1use std::{
7 collections::HashMap,
8 fmt::{self, Formatter},
9 sync::{Arc, LazyLock, Mutex},
10 time::Duration,
11};
12
13use arrow_array::RecordBatch;
14use arrow_schema::Schema as ArrowSchema;
15use datafusion::{
16 catalog::streaming::StreamingTable,
17 dataframe::DataFrame,
18 execution::{
19 context::{SessionConfig, SessionContext},
20 disk_manager::DiskManagerBuilder,
21 memory_pool::FairSpillPool,
22 runtime_env::RuntimeEnvBuilder,
23 TaskContext,
24 },
25 physical_plan::{
26 analyze::AnalyzeExec,
27 display::DisplayableExecutionPlan,
28 execution_plan::{Boundedness, CardinalityEffect, EmissionType},
29 stream::RecordBatchStreamAdapter,
30 streaming::PartitionStream,
31 DisplayAs, DisplayFormatType, ExecutionPlan, PlanProperties, SendableRecordBatchStream,
32 },
33};
34use datafusion_common::{DataFusionError, Statistics};
35use datafusion_physical_expr::{EquivalenceProperties, Partitioning};
36
37use futures::{stream, StreamExt};
38use lance_arrow::SchemaExt;
39use lance_core::{
40 utils::{
41 futures::FinallyStreamExt,
42 tracing::{StreamTracingExt, EXECUTION_PLAN_RUN, TRACE_EXECUTION},
43 },
44 Error, Result,
45};
46use log::{debug, info, warn};
47use snafu::location;
48use tracing::Span;
49
50use crate::udf::register_functions;
51use crate::{
52 chunker::StrictBatchSizeStream,
53 utils::{
54 MetricsExt, BYTES_READ_METRIC, INDEX_COMPARISONS_METRIC, INDICES_LOADED_METRIC,
55 IOPS_METRIC, PARTS_LOADED_METRIC, REQUESTS_METRIC,
56 },
57};
58
59pub struct OneShotExec {
67 stream: Mutex<Option<SendableRecordBatchStream>>,
68 schema: Arc<ArrowSchema>,
71 properties: PlanProperties,
72}
73
74impl OneShotExec {
75 pub fn new(stream: SendableRecordBatchStream) -> Self {
77 let schema = stream.schema();
78 Self {
79 stream: Mutex::new(Some(stream)),
80 schema: schema.clone(),
81 properties: PlanProperties::new(
82 EquivalenceProperties::new(schema),
83 Partitioning::RoundRobinBatch(1),
84 EmissionType::Incremental,
85 Boundedness::Bounded,
86 ),
87 }
88 }
89
90 pub fn from_batch(batch: RecordBatch) -> Self {
91 let schema = batch.schema();
92 let stream = Box::pin(RecordBatchStreamAdapter::new(
93 schema,
94 stream::iter(vec![Ok(batch)]),
95 ));
96 Self::new(stream)
97 }
98}
99
100impl std::fmt::Debug for OneShotExec {
101 fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
102 let stream = self.stream.lock().unwrap();
103 f.debug_struct("OneShotExec")
104 .field("exhausted", &stream.is_none())
105 .field("schema", self.schema.as_ref())
106 .finish()
107 }
108}
109
110impl DisplayAs for OneShotExec {
111 fn fmt_as(
112 &self,
113 t: datafusion::physical_plan::DisplayFormatType,
114 f: &mut std::fmt::Formatter,
115 ) -> std::fmt::Result {
116 let stream = self.stream.lock().unwrap();
117 let exhausted = if stream.is_some() { "" } else { "EXHAUSTED" };
118 let columns = self
119 .schema
120 .field_names()
121 .iter()
122 .cloned()
123 .cloned()
124 .collect::<Vec<_>>();
125 match t {
126 DisplayFormatType::Default | DisplayFormatType::Verbose => {
127 write!(
128 f,
129 "OneShotStream: {}columns=[{}]",
130 exhausted,
131 columns.join(",")
132 )
133 }
134 DisplayFormatType::TreeRender => {
135 write!(
136 f,
137 "OneShotStream\nexhausted={}\ncolumns=[{}]",
138 exhausted,
139 columns.join(",")
140 )
141 }
142 }
143 }
144}
145
146impl ExecutionPlan for OneShotExec {
147 fn name(&self) -> &str {
148 "OneShotExec"
149 }
150
151 fn as_any(&self) -> &dyn std::any::Any {
152 self
153 }
154
155 fn schema(&self) -> arrow_schema::SchemaRef {
156 self.schema.clone()
157 }
158
159 fn children(&self) -> Vec<&Arc<dyn ExecutionPlan>> {
160 vec![]
161 }
162
163 fn with_new_children(
164 self: Arc<Self>,
165 children: Vec<Arc<dyn ExecutionPlan>>,
166 ) -> datafusion_common::Result<Arc<dyn ExecutionPlan>> {
167 if !children.is_empty() {
169 return Err(datafusion_common::DataFusionError::Internal(
170 "OneShotExec does not support children".to_string(),
171 ));
172 }
173 Ok(self)
174 }
175
176 fn execute(
177 &self,
178 _partition: usize,
179 _context: Arc<datafusion::execution::TaskContext>,
180 ) -> datafusion_common::Result<SendableRecordBatchStream> {
181 let stream = self
182 .stream
183 .lock()
184 .map_err(|err| DataFusionError::Execution(err.to_string()))?
185 .take();
186 if let Some(stream) = stream {
187 Ok(stream)
188 } else {
189 Err(DataFusionError::Execution(
190 "OneShotExec has already been executed".to_string(),
191 ))
192 }
193 }
194
195 fn statistics(&self) -> datafusion_common::Result<datafusion_common::Statistics> {
196 Ok(Statistics::new_unknown(&self.schema))
197 }
198
199 fn properties(&self) -> &datafusion::physical_plan::PlanProperties {
200 &self.properties
201 }
202}
203
204struct TracedExec {
205 input: Arc<dyn ExecutionPlan>,
206 properties: PlanProperties,
207 span: Span,
208}
209
210impl TracedExec {
211 pub fn new(input: Arc<dyn ExecutionPlan>, span: Span) -> Self {
212 Self {
213 properties: input.properties().clone(),
214 input,
215 span,
216 }
217 }
218}
219
220impl DisplayAs for TracedExec {
221 fn fmt_as(
222 &self,
223 t: datafusion::physical_plan::DisplayFormatType,
224 f: &mut std::fmt::Formatter,
225 ) -> std::fmt::Result {
226 match t {
227 DisplayFormatType::Default
228 | DisplayFormatType::Verbose
229 | DisplayFormatType::TreeRender => {
230 write!(f, "TracedExec")
231 }
232 }
233 }
234}
235
236impl std::fmt::Debug for TracedExec {
237 fn fmt(&self, f: &mut std::fmt::Formatter) -> std::fmt::Result {
238 write!(f, "TracedExec")
239 }
240}
241impl ExecutionPlan for TracedExec {
242 fn name(&self) -> &str {
243 "TracedExec"
244 }
245
246 fn as_any(&self) -> &dyn std::any::Any {
247 self
248 }
249
250 fn properties(&self) -> &PlanProperties {
251 &self.properties
252 }
253
254 fn children(&self) -> Vec<&Arc<dyn ExecutionPlan>> {
255 vec![&self.input]
256 }
257
258 fn with_new_children(
259 self: Arc<Self>,
260 children: Vec<Arc<dyn ExecutionPlan>>,
261 ) -> datafusion_common::Result<Arc<dyn ExecutionPlan>> {
262 Ok(Arc::new(Self {
263 input: children[0].clone(),
264 properties: self.properties.clone(),
265 span: self.span.clone(),
266 }))
267 }
268
269 fn execute(
270 &self,
271 partition: usize,
272 context: Arc<TaskContext>,
273 ) -> datafusion_common::Result<SendableRecordBatchStream> {
274 let _guard = self.span.enter();
275 let stream = self.input.execute(partition, context)?;
276 let schema = stream.schema();
277 let stream = stream.stream_in_span(self.span.clone());
278 Ok(Box::pin(RecordBatchStreamAdapter::new(schema, stream)))
279 }
280}
281
282pub type ExecutionStatsCallback = Arc<dyn Fn(&ExecutionSummaryCounts) + Send + Sync>;
284
285#[derive(Default, Clone)]
286pub struct LanceExecutionOptions {
287 pub use_spilling: bool,
288 pub mem_pool_size: Option<u64>,
289 pub batch_size: Option<usize>,
290 pub target_partition: Option<usize>,
291 pub execution_stats_callback: Option<ExecutionStatsCallback>,
292}
293
294impl std::fmt::Debug for LanceExecutionOptions {
295 fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
296 f.debug_struct("LanceExecutionOptions")
297 .field("use_spilling", &self.use_spilling)
298 .field("mem_pool_size", &self.mem_pool_size)
299 .field("batch_size", &self.batch_size)
300 .field("target_partition", &self.target_partition)
301 .field(
302 "execution_stats_callback",
303 &self.execution_stats_callback.is_some(),
304 )
305 .finish()
306 }
307}
308
309const DEFAULT_LANCE_MEM_POOL_SIZE: u64 = 100 * 1024 * 1024;
310
311impl LanceExecutionOptions {
312 pub fn mem_pool_size(&self) -> u64 {
313 self.mem_pool_size.unwrap_or_else(|| {
314 std::env::var("LANCE_MEM_POOL_SIZE")
315 .map(|s| match s.parse::<u64>() {
316 Ok(v) => v,
317 Err(e) => {
318 warn!("Failed to parse LANCE_MEM_POOL_SIZE: {}, using default", e);
319 DEFAULT_LANCE_MEM_POOL_SIZE
320 }
321 })
322 .unwrap_or(DEFAULT_LANCE_MEM_POOL_SIZE)
323 })
324 }
325
326 pub fn use_spilling(&self) -> bool {
327 if !self.use_spilling {
328 return false;
329 }
330 std::env::var("LANCE_BYPASS_SPILLING")
331 .map(|_| {
332 info!("Bypassing spilling because LANCE_BYPASS_SPILLING is set");
333 false
334 })
335 .unwrap_or(true)
336 }
337}
338
339pub fn new_session_context(options: &LanceExecutionOptions) -> SessionContext {
340 let mut session_config = SessionConfig::new();
341 let mut runtime_env_builder = RuntimeEnvBuilder::new();
342 if let Some(target_partition) = options.target_partition {
343 session_config = session_config.with_target_partitions(target_partition);
344 }
345 if options.use_spilling() {
346 runtime_env_builder = runtime_env_builder
347 .with_disk_manager_builder(DiskManagerBuilder::default())
348 .with_memory_pool(Arc::new(FairSpillPool::new(
349 options.mem_pool_size() as usize
350 )));
351 }
352 let runtime_env = runtime_env_builder.build_arc().unwrap();
353
354 let ctx = SessionContext::new_with_config_rt(session_config, runtime_env);
355 register_functions(&ctx);
356
357 ctx
358}
359
360static DEFAULT_SESSION_CONTEXT: LazyLock<SessionContext> =
361 LazyLock::new(|| new_session_context(&LanceExecutionOptions::default()));
362
363static DEFAULT_SESSION_CONTEXT_WITH_SPILLING: LazyLock<SessionContext> = LazyLock::new(|| {
364 new_session_context(&LanceExecutionOptions {
365 use_spilling: true,
366 ..Default::default()
367 })
368});
369
370pub fn get_session_context(options: &LanceExecutionOptions) -> SessionContext {
371 if options.mem_pool_size() == DEFAULT_LANCE_MEM_POOL_SIZE && options.target_partition.is_none()
372 {
373 return if options.use_spilling() {
374 DEFAULT_SESSION_CONTEXT_WITH_SPILLING.clone()
375 } else {
376 DEFAULT_SESSION_CONTEXT.clone()
377 };
378 }
379 new_session_context(options)
380}
381
382fn get_task_context(
383 session_ctx: &SessionContext,
384 options: &LanceExecutionOptions,
385) -> Arc<TaskContext> {
386 let mut state = session_ctx.state();
387 if let Some(batch_size) = options.batch_size.as_ref() {
388 state.config_mut().options_mut().execution.batch_size = *batch_size;
389 }
390
391 state.task_ctx()
392}
393
394#[derive(Default, Clone, Debug, PartialEq, Eq)]
395pub struct ExecutionSummaryCounts {
396 pub iops: usize,
398 pub requests: usize,
401 pub bytes_read: usize,
403 pub indices_loaded: usize,
405 pub parts_loaded: usize,
407 pub index_comparisons: usize,
409 pub all_counts: HashMap<String, usize>,
412}
413
414fn visit_node(node: &dyn ExecutionPlan, counts: &mut ExecutionSummaryCounts) {
415 if let Some(metrics) = node.metrics() {
416 for (metric_name, count) in metrics.iter_counts() {
417 match metric_name.as_ref() {
418 IOPS_METRIC => counts.iops += count.value(),
419 REQUESTS_METRIC => counts.requests += count.value(),
420 BYTES_READ_METRIC => counts.bytes_read += count.value(),
421 INDICES_LOADED_METRIC => counts.indices_loaded += count.value(),
422 PARTS_LOADED_METRIC => counts.parts_loaded += count.value(),
423 INDEX_COMPARISONS_METRIC => counts.index_comparisons += count.value(),
424 _ => {
425 let existing = counts
426 .all_counts
427 .entry(metric_name.as_ref().to_string())
428 .or_insert(0);
429 *existing += count.value();
430 }
431 }
432 }
433 for (metric_name, gauge) in metrics.iter_gauges() {
435 match metric_name.as_ref() {
436 IOPS_METRIC => counts.iops += gauge.value(),
437 REQUESTS_METRIC => counts.requests += gauge.value(),
438 BYTES_READ_METRIC => counts.bytes_read += gauge.value(),
439 _ => {}
440 }
441 }
442 }
443 for child in node.children() {
444 visit_node(child.as_ref(), counts);
445 }
446}
447
448fn report_plan_summary_metrics(plan: &dyn ExecutionPlan, options: &LanceExecutionOptions) {
449 let output_rows = plan
450 .metrics()
451 .map(|m| m.output_rows().unwrap_or(0))
452 .unwrap_or(0);
453 let mut counts = ExecutionSummaryCounts::default();
454 visit_node(plan, &mut counts);
455 tracing::info!(
456 target: TRACE_EXECUTION,
457 r#type = EXECUTION_PLAN_RUN,
458 plan_summary = display_plan_one_liner(plan),
459 output_rows,
460 iops = counts.iops,
461 requests = counts.requests,
462 bytes_read = counts.bytes_read,
463 indices_loaded = counts.indices_loaded,
464 parts_loaded = counts.parts_loaded,
465 index_comparisons = counts.index_comparisons,
466 );
467 if let Some(callback) = options.execution_stats_callback.as_ref() {
468 callback(&counts);
469 }
470}
471
472fn display_plan_one_liner(plan: &dyn ExecutionPlan) -> String {
479 let mut output = String::new();
480
481 display_plan_one_liner_impl(plan, &mut output);
482
483 output
484}
485
486fn display_plan_one_liner_impl(plan: &dyn ExecutionPlan, output: &mut String) {
487 let name = plan.name().trim_end_matches("Exec");
489 output.push_str(name);
490
491 let children = plan.children();
492 if !children.is_empty() {
493 output.push('(');
494 for (i, child) in children.iter().enumerate() {
495 if i > 0 {
496 output.push(',');
497 }
498 display_plan_one_liner_impl(child.as_ref(), output);
499 }
500 output.push(')');
501 }
502}
503
504pub fn execute_plan(
508 plan: Arc<dyn ExecutionPlan>,
509 options: LanceExecutionOptions,
510) -> Result<SendableRecordBatchStream> {
511 debug!(
512 "Executing plan:\n{}",
513 DisplayableExecutionPlan::new(plan.as_ref()).indent(true)
514 );
515
516 let session_ctx = get_session_context(&options);
517
518 assert_eq!(plan.properties().partitioning.partition_count(), 1);
521 let stream = plan.execute(0, get_task_context(&session_ctx, &options))?;
522
523 let schema = stream.schema();
524 let stream = stream.finally(move || {
525 report_plan_summary_metrics(plan.as_ref(), &options);
526 });
527 Ok(Box::pin(RecordBatchStreamAdapter::new(schema, stream)))
528}
529
530pub async fn analyze_plan(
531 plan: Arc<dyn ExecutionPlan>,
532 options: LanceExecutionOptions,
533) -> Result<String> {
534 let plan = Arc::new(TracedExec::new(plan, Span::current()));
537
538 let schema = plan.schema();
539 let analyze = Arc::new(AnalyzeExec::new(true, true, plan, schema));
540
541 let session_ctx = get_session_context(&options);
542 assert_eq!(analyze.properties().partitioning.partition_count(), 1);
543 let mut stream = analyze
544 .execute(0, get_task_context(&session_ctx, &options))
545 .map_err(|err| {
546 Error::io(
547 format!("Failed to execute analyze plan: {}", err),
548 location!(),
549 )
550 })?;
551
552 while (stream.next().await).is_some() {}
554
555 let result = format_plan(analyze);
556 Ok(result)
557}
558
559pub fn format_plan(plan: Arc<dyn ExecutionPlan>) -> String {
560 struct CalculateVisitor {
562 highest_index: usize,
563 index_to_cumulative_cpu: HashMap<usize, usize>,
564 }
565 impl CalculateVisitor {
566 fn calculate_cumulative_cpu(&mut self, plan: &Arc<dyn ExecutionPlan>) -> usize {
567 self.highest_index += 1;
568 let plan_index = self.highest_index;
569 let elapsed_cpu: usize = match plan.metrics() {
570 Some(metrics) => metrics.elapsed_compute().unwrap_or_default(),
571 None => 0,
572 };
573 let mut cumulative_cpu = elapsed_cpu;
574 for child in plan.children() {
575 cumulative_cpu += self.calculate_cumulative_cpu(child);
576 }
577 self.index_to_cumulative_cpu
578 .insert(plan_index, cumulative_cpu);
579 cumulative_cpu
580 }
581 }
582
583 struct PrintVisitor {
585 highest_index: usize,
586 indent: usize,
587 }
588 impl PrintVisitor {
589 fn write_output(
590 &mut self,
591 plan: &Arc<dyn ExecutionPlan>,
592 f: &mut Formatter,
593 calcs: &CalculateVisitor,
594 ) -> std::fmt::Result {
595 self.highest_index += 1;
596 write!(f, "{:indent$}", "", indent = self.indent * 2)?;
597 plan.fmt_as(datafusion::physical_plan::DisplayFormatType::Verbose, f)?;
598 if let Some(metrics) = plan.metrics() {
599 let metrics = metrics
600 .aggregate_by_name()
601 .sorted_for_display()
602 .timestamps_removed();
603
604 write!(f, ", metrics=[{metrics}]")?;
605 } else {
606 write!(f, ", metrics=[]")?;
607 }
608 let cumulative_cpu = calcs
609 .index_to_cumulative_cpu
610 .get(&self.highest_index)
611 .unwrap();
612 let cumulative_cpu_duration = Duration::from_nanos((*cumulative_cpu) as u64);
613 write!(f, ", cumulative_cpu={cumulative_cpu_duration:?}")?;
614 writeln!(f)?;
615 self.indent += 1;
616 for child in plan.children() {
617 self.write_output(child, f, calcs)?;
618 }
619 self.indent -= 1;
620 std::fmt::Result::Ok(())
621 }
622 }
623 struct PrintWrapper {
625 plan: Arc<dyn ExecutionPlan>,
626 }
627 impl fmt::Display for PrintWrapper {
628 fn fmt(&self, f: &mut Formatter) -> std::fmt::Result {
629 let mut calcs = CalculateVisitor {
630 highest_index: 0,
631 index_to_cumulative_cpu: HashMap::new(),
632 };
633 calcs.calculate_cumulative_cpu(&self.plan);
634 let mut prints = PrintVisitor {
635 highest_index: 0,
636 indent: 0,
637 };
638 prints.write_output(&self.plan, f, &calcs)
639 }
640 }
641 let wrapper = PrintWrapper { plan };
642 format!("{}", wrapper)
643}
644
645pub trait SessionContextExt {
646 fn read_one_shot(
650 &self,
651 data: SendableRecordBatchStream,
652 ) -> datafusion::common::Result<DataFrame>;
653}
654
655struct OneShotPartitionStream {
656 data: Arc<Mutex<Option<SendableRecordBatchStream>>>,
657 schema: Arc<ArrowSchema>,
658}
659
660impl std::fmt::Debug for OneShotPartitionStream {
661 fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
662 let data = self.data.lock().unwrap();
663 f.debug_struct("OneShotPartitionStream")
664 .field("exhausted", &data.is_none())
665 .field("schema", self.schema.as_ref())
666 .finish()
667 }
668}
669
670impl OneShotPartitionStream {
671 fn new(data: SendableRecordBatchStream) -> Self {
672 let schema = data.schema();
673 Self {
674 data: Arc::new(Mutex::new(Some(data))),
675 schema,
676 }
677 }
678}
679
680impl PartitionStream for OneShotPartitionStream {
681 fn schema(&self) -> &arrow_schema::SchemaRef {
682 &self.schema
683 }
684
685 fn execute(&self, _ctx: Arc<TaskContext>) -> SendableRecordBatchStream {
686 let mut stream = self.data.lock().unwrap();
687 stream
688 .take()
689 .expect("Attempt to consume a one shot dataframe multiple times")
690 }
691}
692
693impl SessionContextExt for SessionContext {
694 fn read_one_shot(
695 &self,
696 data: SendableRecordBatchStream,
697 ) -> datafusion::common::Result<DataFrame> {
698 let schema = data.schema();
699 let part_stream = Arc::new(OneShotPartitionStream::new(data));
700 let provider = StreamingTable::try_new(schema, vec![part_stream])?;
701 self.read_table(Arc::new(provider))
702 }
703}
704
705#[derive(Clone, Debug)]
706pub struct StrictBatchSizeExec {
707 input: Arc<dyn ExecutionPlan>,
708 batch_size: usize,
709}
710
711impl StrictBatchSizeExec {
712 pub fn new(input: Arc<dyn ExecutionPlan>, batch_size: usize) -> Self {
713 Self { input, batch_size }
714 }
715}
716
717impl DisplayAs for StrictBatchSizeExec {
718 fn fmt_as(
719 &self,
720 _t: datafusion::physical_plan::DisplayFormatType,
721 f: &mut std::fmt::Formatter,
722 ) -> std::fmt::Result {
723 write!(f, "StrictBatchSizeExec")
724 }
725}
726
727impl ExecutionPlan for StrictBatchSizeExec {
728 fn name(&self) -> &str {
729 "StrictBatchSizeExec"
730 }
731
732 fn as_any(&self) -> &dyn std::any::Any {
733 self
734 }
735
736 fn properties(&self) -> &PlanProperties {
737 self.input.properties()
738 }
739
740 fn children(&self) -> Vec<&Arc<dyn ExecutionPlan>> {
741 vec![&self.input]
742 }
743
744 fn with_new_children(
745 self: Arc<Self>,
746 children: Vec<Arc<dyn ExecutionPlan>>,
747 ) -> datafusion_common::Result<Arc<dyn ExecutionPlan>> {
748 Ok(Arc::new(Self {
749 input: children[0].clone(),
750 batch_size: self.batch_size,
751 }))
752 }
753
754 fn execute(
755 &self,
756 partition: usize,
757 context: Arc<TaskContext>,
758 ) -> datafusion_common::Result<SendableRecordBatchStream> {
759 let stream = self.input.execute(partition, context)?;
760 let schema = stream.schema();
761 let stream = StrictBatchSizeStream::new(stream, self.batch_size);
762 Ok(Box::pin(RecordBatchStreamAdapter::new(schema, stream)))
763 }
764
765 fn maintains_input_order(&self) -> Vec<bool> {
766 vec![true]
767 }
768
769 fn benefits_from_input_partitioning(&self) -> Vec<bool> {
770 vec![false]
771 }
772
773 fn partition_statistics(
774 &self,
775 partition: Option<usize>,
776 ) -> datafusion_common::Result<Statistics> {
777 self.input.partition_statistics(partition)
778 }
779
780 fn cardinality_effect(&self) -> CardinalityEffect {
781 CardinalityEffect::Equal
782 }
783
784 fn supports_limit_pushdown(&self) -> bool {
785 true
786 }
787}