-
Notifications
You must be signed in to change notification settings - Fork 1.3k
/
Copy pathcontext.rs
2523 lines (2248 loc) · 86.9 KB
/
context.rs
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
// Licensed to the Apache Software Foundation (ASF) under one
// or more contributor license agreements. See the NOTICE file
// distributed with this work for additional information
// regarding copyright ownership. The ASF licenses this file
// to you under the Apache License, Version 2.0 (the
// "License"); you may not use this file except in compliance
// with the License. You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing,
// software distributed under the License is distributed on an
// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
// KIND, either express or implied. See the License for the
// specific language governing permissions and limitations
// under the License.
//! SessionContext contains methods for registering data sources and executing queries
use crate::{
catalog::{
catalog::{CatalogList, MemoryCatalogList},
information_schema::CatalogWithInformationSchema,
},
datasource::listing::{ListingOptions, ListingTable},
datasource::{
file_format::{
avro::AvroFormat, csv::CsvFormat, json::JsonFormat, parquet::ParquetFormat,
FileFormat,
},
MemTable, ViewTable,
},
logical_expr::{PlanType, ToStringifiedPlan},
optimizer::optimizer::Optimizer,
physical_optimizer::{
aggregate_statistics::AggregateStatistics,
hash_build_probe_order::HashBuildProbeOrder, optimizer::PhysicalOptimizerRule,
},
};
pub use datafusion_physical_expr::execution_props::ExecutionProps;
use datafusion_physical_expr::var_provider::is_system_variables;
use parking_lot::RwLock;
use std::str::FromStr;
use std::sync::Arc;
use std::{
any::{Any, TypeId},
hash::{BuildHasherDefault, Hasher},
string::String,
};
use std::{
collections::{HashMap, HashSet},
fmt::Debug,
};
use arrow::datatypes::{DataType, SchemaRef};
use arrow::record_batch::RecordBatch;
use crate::catalog::{
catalog::{CatalogProvider, MemoryCatalogProvider},
schema::{MemorySchemaProvider, SchemaProvider},
};
use crate::dataframe::DataFrame;
use crate::datasource::{
listing::{ListingTableConfig, ListingTableUrl},
provider_as_source, TableProvider,
};
use crate::error::{DataFusionError, Result};
use crate::logical_expr::{
CreateCatalog, CreateCatalogSchema, CreateExternalTable, CreateMemoryTable,
CreateView, DropTable, DropView, Explain, LogicalPlan, LogicalPlanBuilder,
TableSource, TableType, UNNAMED_TABLE,
};
use crate::optimizer::optimizer::{OptimizerConfig, OptimizerRule};
use datafusion_sql::{ResolvedTableReference, TableReference};
use crate::physical_optimizer::coalesce_batches::CoalesceBatches;
use crate::physical_optimizer::merge_exec::AddCoalescePartitionsExec;
use crate::physical_optimizer::repartition::Repartition;
use crate::config::{
ConfigOptions, OPT_BATCH_SIZE, OPT_COALESCE_BATCHES, OPT_COALESCE_TARGET_BATCH_SIZE,
OPT_FILTER_NULL_JOIN_KEYS, OPT_OPTIMIZER_SKIP_FAILED_RULES,
};
use crate::datasource::file_format::file_type::{FileCompressionType, FileType};
use crate::execution::{runtime_env::RuntimeEnv, FunctionRegistry};
use crate::physical_plan::file_format::{plan_to_csv, plan_to_json, plan_to_parquet};
use crate::physical_plan::planner::DefaultPhysicalPlanner;
use crate::physical_plan::udaf::AggregateUDF;
use crate::physical_plan::udf::ScalarUDF;
use crate::physical_plan::ExecutionPlan;
use crate::physical_plan::PhysicalPlanner;
use crate::variable::{VarProvider, VarType};
use async_trait::async_trait;
use chrono::{DateTime, Utc};
use datafusion_common::ScalarValue;
use datafusion_sql::{
parser::DFParser,
planner::{ContextProvider, SqlToRel},
};
use parquet::file::properties::WriterProperties;
use uuid::Uuid;
use super::options::{
AvroReadOptions, CsvReadOptions, NdJsonReadOptions, ParquetReadOptions,
};
/// The default catalog name - this impacts what SQL queries use if not specified
const DEFAULT_CATALOG: &str = "datafusion";
/// The default schema name - this impacts what SQL queries use if not specified
const DEFAULT_SCHEMA: &str = "public";
/// SessionContext is the main interface for executing queries with DataFusion. It stands for
/// the connection between user and DataFusion/Ballista cluster.
/// The context provides the following functionality
///
/// * Create DataFrame from a CSV or Parquet data source.
/// * Register a CSV or Parquet data source as a table that can be referenced from a SQL query.
/// * Register a custom data source that can be referenced from a SQL query.
/// * Execution a SQL query
///
/// The following example demonstrates how to use the context to execute a query against a CSV
/// data source using the DataFrame API:
///
/// ```
/// use datafusion::prelude::*;
/// # use datafusion::error::Result;
/// # #[tokio::main]
/// # async fn main() -> Result<()> {
/// let ctx = SessionContext::new();
/// let df = ctx.read_csv("tests/example.csv", CsvReadOptions::new()).await?;
/// let df = df.filter(col("a").lt_eq(col("b")))?
/// .aggregate(vec![col("a")], vec![min(col("b"))])?
/// .limit(0, Some(100))?;
/// let results = df.collect();
/// # Ok(())
/// # }
/// ```
///
/// The following example demonstrates how to execute the same query using SQL:
///
/// ```
/// use datafusion::prelude::*;
///
/// # use datafusion::error::Result;
/// # #[tokio::main]
/// # async fn main() -> Result<()> {
/// let mut ctx = SessionContext::new();
/// ctx.register_csv("example", "tests/example.csv", CsvReadOptions::new()).await?;
/// let results = ctx.sql("SELECT a, MIN(b) FROM example GROUP BY a LIMIT 100").await?;
/// # Ok(())
/// # }
/// ```
#[derive(Clone)]
pub struct SessionContext {
/// Uuid for the session
session_id: String,
/// Session start time
pub session_start_time: DateTime<Utc>,
/// Shared session state for the session
pub state: Arc<RwLock<SessionState>>,
}
impl Default for SessionContext {
fn default() -> Self {
Self::new()
}
}
impl SessionContext {
/// Creates a new execution context using a default session configuration.
pub fn new() -> Self {
Self::with_config(SessionConfig::new())
}
/// Creates a new session context using the provided session configuration.
pub fn with_config(config: SessionConfig) -> Self {
let runtime = Arc::new(RuntimeEnv::default());
Self::with_config_rt(config, runtime)
}
/// Creates a new session context using the provided configuration and RuntimeEnv.
pub fn with_config_rt(config: SessionConfig, runtime: Arc<RuntimeEnv>) -> Self {
let state = SessionState::with_config_rt(config, runtime);
Self {
session_id: state.session_id.clone(),
session_start_time: chrono::Utc::now(),
state: Arc::new(RwLock::new(state)),
}
}
/// Creates a new session context using the provided session state.
pub fn with_state(state: SessionState) -> Self {
Self {
session_id: state.session_id.clone(),
session_start_time: chrono::Utc::now(),
state: Arc::new(RwLock::new(state)),
}
}
/// Registers the [`RecordBatch`] as the specified table name
pub fn register_batch(
&self,
table_name: &str,
batch: RecordBatch,
) -> Result<Option<Arc<dyn TableProvider>>> {
let table = MemTable::try_new(batch.schema(), vec![vec![batch]])?;
self.register_table(table_name, Arc::new(table))
}
/// Return the [RuntimeEnv] used to run queries with this [SessionContext]
pub fn runtime_env(&self) -> Arc<RuntimeEnv> {
self.state.read().runtime_env.clone()
}
/// Return the session_id of this Session
pub fn session_id(&self) -> String {
self.session_id.clone()
}
/// Return a copied version of config for this Session
pub fn copied_config(&self) -> SessionConfig {
self.state.read().config.clone()
}
/// Creates a [`DataFrame`] that will execute a SQL query.
///
/// This method is `async` because queries of type `CREATE EXTERNAL TABLE`
/// might require the schema to be inferred.
pub async fn sql(&self, sql: &str) -> Result<Arc<DataFrame>> {
let plan = self.create_logical_plan(sql)?;
match plan {
LogicalPlan::CreateExternalTable(cmd) => match cmd.file_type.as_str() {
"PARQUET" | "CSV" | "JSON" | "AVRO" => {
self.create_listing_table(&cmd).await
}
_ => self.create_custom_table(&cmd).await,
},
LogicalPlan::CreateMemoryTable(CreateMemoryTable {
name,
input,
if_not_exists,
or_replace,
}) => {
let table = self.table(name.as_str());
match (if_not_exists, or_replace, table) {
(true, false, Ok(_)) => self.return_empty_dataframe(),
(false, true, Ok(_)) => {
self.deregister_table(name.as_str())?;
let physical =
Arc::new(DataFrame::new(self.state.clone(), &input));
let batches: Vec<_> = physical.collect_partitioned().await?;
let table = Arc::new(MemTable::try_new(
Arc::new(input.schema().as_ref().into()),
batches,
)?);
self.register_table(name.as_str(), table)?;
self.return_empty_dataframe()
}
(true, true, Ok(_)) => Err(DataFusionError::Internal(
"'IF NOT EXISTS' cannot coexist with 'REPLACE'".to_string(),
)),
(_, _, Err(_)) => {
let physical =
Arc::new(DataFrame::new(self.state.clone(), &input));
let batches: Vec<_> = physical.collect_partitioned().await?;
let table = Arc::new(MemTable::try_new(
Arc::new(input.schema().as_ref().into()),
batches,
)?);
self.register_table(name.as_str(), table)?;
self.return_empty_dataframe()
}
(false, false, Ok(_)) => Err(DataFusionError::Execution(format!(
"Table '{:?}' already exists",
name
))),
}
}
LogicalPlan::CreateView(CreateView {
name,
input,
or_replace,
definition,
}) => {
let view = self.table(name.as_str());
match (or_replace, view) {
(true, Ok(_)) => {
self.deregister_table(name.as_str())?;
let table =
Arc::new(ViewTable::try_new((*input).clone(), definition)?);
self.register_table(name.as_str(), table)?;
self.return_empty_dataframe()
}
(_, Err(_)) => {
let table =
Arc::new(ViewTable::try_new((*input).clone(), definition)?);
self.register_table(name.as_str(), table)?;
self.return_empty_dataframe()
}
(false, Ok(_)) => Err(DataFusionError::Execution(format!(
"Table '{:?}' already exists",
name
))),
}
}
LogicalPlan::DropTable(DropTable {
name, if_exists, ..
}) => {
let result = self.find_and_deregister(name.as_str(), TableType::Base);
match (result, if_exists) {
(Ok(true), _) => self.return_empty_dataframe(),
(_, true) => self.return_empty_dataframe(),
(_, _) => Err(DataFusionError::Execution(format!(
"Table {:?} doesn't exist.",
name
))),
}
}
LogicalPlan::DropView(DropView {
name, if_exists, ..
}) => {
let result = self.find_and_deregister(name.as_str(), TableType::View);
match (result, if_exists) {
(Ok(true), _) => self.return_empty_dataframe(),
(_, true) => self.return_empty_dataframe(),
(_, _) => Err(DataFusionError::Execution(format!(
"View {:?} doesn't exist.",
name
))),
}
}
LogicalPlan::CreateCatalogSchema(CreateCatalogSchema {
schema_name,
if_not_exists,
..
}) => {
// sqlparser doesnt accept database / catalog as parameter to CREATE SCHEMA
// so for now, we default to default catalog
let tokens: Vec<&str> = schema_name.split('.').collect();
let (catalog, schema_name) = match tokens.len() {
1 => Ok((DEFAULT_CATALOG, schema_name.as_str())),
2 => Ok((tokens[0], tokens[1])),
_ => Err(DataFusionError::Execution(format!(
"Unable to parse catalog from {}",
schema_name
))),
}?;
let catalog = self.catalog(catalog).ok_or_else(|| {
DataFusionError::Execution(format!(
"Missing '{}' catalog",
DEFAULT_CATALOG
))
})?;
let schema = catalog.schema(schema_name);
match (if_not_exists, schema) {
(true, Some(_)) => self.return_empty_dataframe(),
(true, None) | (false, None) => {
let schema = Arc::new(MemorySchemaProvider::new());
catalog.register_schema(schema_name, schema)?;
self.return_empty_dataframe()
}
(false, Some(_)) => Err(DataFusionError::Execution(format!(
"Schema '{:?}' already exists",
schema_name
))),
}
}
LogicalPlan::CreateCatalog(CreateCatalog {
catalog_name,
if_not_exists,
..
}) => {
let catalog = self.catalog(catalog_name.as_str());
match (if_not_exists, catalog) {
(true, Some(_)) => self.return_empty_dataframe(),
(true, None) | (false, None) => {
let new_catalog = Arc::new(MemoryCatalogProvider::new());
self.state
.write()
.catalog_list
.register_catalog(catalog_name, new_catalog);
self.return_empty_dataframe()
}
(false, Some(_)) => Err(DataFusionError::Execution(format!(
"Catalog '{:?}' already exists",
catalog_name
))),
}
}
plan => Ok(Arc::new(DataFrame::new(self.state.clone(), &plan))),
}
}
// return an empty dataframe
fn return_empty_dataframe(&self) -> Result<Arc<DataFrame>> {
let plan = LogicalPlanBuilder::empty(false).build()?;
Ok(Arc::new(DataFrame::new(self.state.clone(), &plan)))
}
async fn create_custom_table(
&self,
cmd: &CreateExternalTable,
) -> Result<Arc<DataFrame>> {
let state = self.state.read().clone();
let factory = &state
.runtime_env
.table_factories
.get(&cmd.file_type)
.ok_or_else(|| {
DataFusionError::Execution(format!(
"Unable to find factory for {}",
cmd.file_type
))
})?;
let table = (*factory)
.create(cmd.name.as_str(), cmd.location.as_str())
.await?;
self.register_table(cmd.name.as_str(), table)?;
let plan = LogicalPlanBuilder::empty(false).build()?;
Ok(Arc::new(DataFrame::new(self.state.clone(), &plan)))
}
async fn create_listing_table(
&self,
cmd: &CreateExternalTable,
) -> Result<Arc<DataFrame>> {
let file_compression_type =
match FileCompressionType::from_str(cmd.file_compression_type.as_str()) {
Ok(t) => t,
Err(_) => Err(DataFusionError::Execution(
"Only known FileCompressionTypes can be ListingTables!".to_string(),
))?,
};
let file_type = match FileType::from_str(cmd.file_type.as_str()) {
Ok(t) => t,
Err(_) => Err(DataFusionError::Execution(
"Only known FileTypes can be ListingTables!".to_string(),
))?,
};
let file_extension =
file_type.get_ext_with_compression(file_compression_type.to_owned())?;
let file_format: Arc<dyn FileFormat> = match file_type {
FileType::CSV => Arc::new(
CsvFormat::default()
.with_has_header(cmd.has_header)
.with_delimiter(cmd.delimiter as u8)
.with_file_compression_type(file_compression_type),
),
FileType::PARQUET => Arc::new(ParquetFormat::default()),
FileType::AVRO => Arc::new(AvroFormat::default()),
FileType::JSON => Arc::new(
JsonFormat::default().with_file_compression_type(file_compression_type),
),
};
let table = self.table(cmd.name.as_str());
match (cmd.if_not_exists, table) {
(true, Ok(_)) => self.return_empty_dataframe(),
(_, Err(_)) => {
// TODO make schema in CreateExternalTable optional instead of empty
let provided_schema = if cmd.schema.fields().is_empty() {
None
} else {
Some(Arc::new(cmd.schema.as_ref().to_owned().into()))
};
let options = ListingOptions {
format: file_format,
collect_stat: self.copied_config().collect_statistics,
file_extension: file_extension.to_owned(),
target_partitions: self.copied_config().target_partitions,
table_partition_cols: cmd.table_partition_cols.clone(),
};
self.register_listing_table(
cmd.name.as_str(),
cmd.location.clone(),
options,
provided_schema,
cmd.definition.clone(),
)
.await?;
self.return_empty_dataframe()
}
(false, Ok(_)) => Err(DataFusionError::Execution(format!(
"Table '{:?}' already exists",
cmd.name
))),
}
}
fn find_and_deregister<'a>(
&self,
table_ref: impl Into<TableReference<'a>>,
table_type: TableType,
) -> Result<bool> {
let table_ref = table_ref.into();
let table_provider = self
.state
.read()
.schema_for_ref(table_ref)?
.table(table_ref.table());
if let Some(table_provider) = table_provider {
if table_provider.table_type() == table_type {
self.deregister_table(table_ref)?;
return Ok(true);
}
}
Ok(false)
}
/// Creates a logical plan.
///
/// This function is intended for internal use and should not be called directly.
pub fn create_logical_plan(&self, sql: &str) -> Result<LogicalPlan> {
let mut statements = DFParser::parse_sql(sql)?;
if statements.len() != 1 {
return Err(DataFusionError::NotImplemented(
"The context currently only supports a single SQL statement".to_string(),
));
}
// create a query planner
let state = self.state.read().clone();
let query_planner = SqlToRel::new(&state);
query_planner.statement_to_plan(statements.pop_front().unwrap())
}
/// Registers a variable provider within this context.
pub fn register_variable(
&mut self,
variable_type: VarType,
provider: Arc<dyn VarProvider + Send + Sync>,
) {
self.state
.write()
.execution_props
.add_var_provider(variable_type, provider);
}
/// Registers a scalar UDF within this context.
///
/// Note in SQL queries, function names are looked up using
/// lowercase unless the query uses quotes. For example,
///
/// `SELECT MY_FUNC(x)...` will look for a function named `"my_func"`
/// `SELECT "my_FUNC"(x)` will look for a function named `"my_FUNC"`
pub fn register_udf(&mut self, f: ScalarUDF) {
self.state
.write()
.scalar_functions
.insert(f.name.clone(), Arc::new(f));
}
/// Registers an aggregate UDF within this context.
///
/// Note in SQL queries, aggregate names are looked up using
/// lowercase unless the query uses quotes. For example,
///
/// `SELECT MY_UDAF(x)...` will look for an aggregate named `"my_udaf"`
/// `SELECT "my_UDAF"(x)` will look for an aggregate named `"my_UDAF"`
pub fn register_udaf(&mut self, f: AggregateUDF) {
self.state
.write()
.aggregate_functions
.insert(f.name.clone(), Arc::new(f));
}
/// Creates a [`DataFrame`] for reading an Avro data source.
pub async fn read_avro(
&self,
table_path: impl AsRef<str>,
options: AvroReadOptions<'_>,
) -> Result<Arc<DataFrame>> {
let table_path = ListingTableUrl::parse(table_path)?;
let target_partitions = self.copied_config().target_partitions;
let listing_options = options.to_listing_options(target_partitions);
let resolved_schema = match options.schema {
Some(s) => s,
None => {
listing_options
.infer_schema(&self.state(), &table_path)
.await?
}
};
let config = ListingTableConfig::new(table_path)
.with_listing_options(listing_options)
.with_schema(resolved_schema);
let provider = ListingTable::try_new(config)?;
self.read_table(Arc::new(provider))
}
/// Creates a [`DataFrame`] for reading an Json data source.
pub async fn read_json(
&mut self,
table_path: impl AsRef<str>,
options: NdJsonReadOptions<'_>,
) -> Result<Arc<DataFrame>> {
let table_path = ListingTableUrl::parse(table_path)?;
let target_partitions = self.copied_config().target_partitions;
let listing_options = options.to_listing_options(target_partitions);
let resolved_schema = match options.schema {
Some(s) => s,
None => {
listing_options
.infer_schema(&self.state(), &table_path)
.await?
}
};
let config = ListingTableConfig::new(table_path)
.with_listing_options(listing_options)
.with_schema(resolved_schema);
let provider = ListingTable::try_new(config)?;
self.read_table(Arc::new(provider))
}
/// Creates an empty DataFrame.
pub fn read_empty(&self) -> Result<Arc<DataFrame>> {
Ok(Arc::new(DataFrame::new(
self.state.clone(),
&LogicalPlanBuilder::empty(true).build()?,
)))
}
/// Creates a [`DataFrame`] for reading a CSV data source.
pub async fn read_csv(
&self,
table_path: impl AsRef<str>,
options: CsvReadOptions<'_>,
) -> Result<Arc<DataFrame>> {
let table_path = ListingTableUrl::parse(table_path)?;
let target_partitions = self.copied_config().target_partitions;
let listing_options = options.to_listing_options(target_partitions);
let resolved_schema = match options.schema {
Some(s) => Arc::new(s.to_owned()),
None => {
listing_options
.infer_schema(&self.state(), &table_path)
.await?
}
};
let config = ListingTableConfig::new(table_path.clone())
.with_listing_options(listing_options)
.with_schema(resolved_schema);
let provider = ListingTable::try_new(config)?;
self.read_table(Arc::new(provider))
}
/// Creates a [`DataFrame`] for reading a Parquet data source.
pub async fn read_parquet(
&self,
table_path: impl AsRef<str>,
options: ParquetReadOptions<'_>,
) -> Result<Arc<DataFrame>> {
let table_path = ListingTableUrl::parse(table_path)?;
let target_partitions = self.copied_config().target_partitions;
let listing_options = options.to_listing_options(target_partitions);
// with parquet we resolve the schema in all cases
let resolved_schema = listing_options
.infer_schema(&self.state(), &table_path)
.await?;
let config = ListingTableConfig::new(table_path)
.with_listing_options(listing_options)
.with_schema(resolved_schema);
let provider = ListingTable::try_new(config)?;
self.read_table(Arc::new(provider))
}
/// Creates a [`DataFrame`] for reading a custom [`TableProvider`].
pub fn read_table(&self, provider: Arc<dyn TableProvider>) -> Result<Arc<DataFrame>> {
Ok(Arc::new(DataFrame::new(
self.state.clone(),
&LogicalPlanBuilder::scan(UNNAMED_TABLE, provider_as_source(provider), None)?
.build()?,
)))
}
/// Creates a [`DataFrame`] for reading a [`RecordBatch`]
pub fn read_batch(&self, batch: RecordBatch) -> Result<Arc<DataFrame>> {
let provider = MemTable::try_new(batch.schema(), vec![vec![batch]])?;
Ok(Arc::new(DataFrame::new(
self.state.clone(),
&LogicalPlanBuilder::scan(
UNNAMED_TABLE,
provider_as_source(Arc::new(provider)),
None,
)?
.build()?,
)))
}
/// Registers a [`ListingTable]` that can assemble multiple files
/// from locations in an [`ObjectStore`] instance into a single
/// table.
///
/// This method is `async` because it might need to resolve the schema.
///
/// [`ObjectStore`]: object_store::ObjectStore
pub async fn register_listing_table(
&self,
name: &str,
table_path: impl AsRef<str>,
options: ListingOptions,
provided_schema: Option<SchemaRef>,
sql: Option<String>,
) -> Result<()> {
let table_path = ListingTableUrl::parse(table_path)?;
let resolved_schema = match provided_schema {
None => options.infer_schema(&self.state(), &table_path).await?,
Some(s) => s,
};
let config = ListingTableConfig::new(table_path)
.with_listing_options(options)
.with_schema(resolved_schema);
let table = ListingTable::try_new(config)?.with_definition(sql);
self.register_table(name, Arc::new(table))?;
Ok(())
}
/// Registers a CSV file as a table which can referenced from SQL
/// statements executed against this context.
pub async fn register_csv(
&self,
name: &str,
table_path: &str,
options: CsvReadOptions<'_>,
) -> Result<()> {
let listing_options =
options.to_listing_options(self.copied_config().target_partitions);
self.register_listing_table(
name,
table_path,
listing_options,
options.schema.map(|s| Arc::new(s.to_owned())),
None,
)
.await?;
Ok(())
}
/// Registers a Json file as a table that it can be referenced
/// from SQL statements executed against this context.
pub async fn register_json(
&self,
name: &str,
table_path: &str,
options: NdJsonReadOptions<'_>,
) -> Result<()> {
let listing_options =
options.to_listing_options(self.copied_config().target_partitions);
self.register_listing_table(
name,
table_path,
listing_options,
options.schema,
None,
)
.await?;
Ok(())
}
/// Registers a Parquet file as a table that can be referenced from SQL
/// statements executed against this context.
pub async fn register_parquet(
&self,
name: &str,
table_path: &str,
options: ParquetReadOptions<'_>,
) -> Result<()> {
let (target_partitions, parquet_pruning) = {
let conf = self.copied_config();
(conf.target_partitions, conf.parquet_pruning)
};
let listing_options = options
.parquet_pruning(parquet_pruning)
.to_listing_options(target_partitions);
self.register_listing_table(name, table_path, listing_options, None, None)
.await?;
Ok(())
}
/// Registers an Avro file as a table that can be referenced from
/// SQL statements executed against this context.
pub async fn register_avro(
&self,
name: &str,
table_path: &str,
options: AvroReadOptions<'_>,
) -> Result<()> {
let listing_options =
options.to_listing_options(self.copied_config().target_partitions);
self.register_listing_table(
name,
table_path,
listing_options,
options.schema,
None,
)
.await?;
Ok(())
}
/// Registers a named catalog using a custom `CatalogProvider` so that
/// it can be referenced from SQL statements executed against this
/// context.
///
/// Returns the [`CatalogProvider`] previously registered for this
/// name, if any
pub fn register_catalog(
&self,
name: impl Into<String>,
catalog: Arc<dyn CatalogProvider>,
) -> Option<Arc<dyn CatalogProvider>> {
let name = name.into();
let information_schema = self.copied_config().information_schema;
let state = self.state.read();
let catalog = if information_schema {
Arc::new(CatalogWithInformationSchema::new(
Arc::downgrade(&state.catalog_list),
Arc::downgrade(&state.config.config_options),
catalog,
))
} else {
catalog
};
state.catalog_list.register_catalog(name, catalog)
}
/// Retrieves a [`CatalogProvider`] instance by name
pub fn catalog(&self, name: &str) -> Option<Arc<dyn CatalogProvider>> {
self.state.read().catalog_list.catalog(name)
}
/// Registers a [`TableProvider`] as a table that can be
/// referenced from SQL statements executed against this context.
///
/// Returns the [`TableProvider`] previously registered for this
/// reference, if any
pub fn register_table<'a>(
&'a self,
table_ref: impl Into<TableReference<'a>>,
provider: Arc<dyn TableProvider>,
) -> Result<Option<Arc<dyn TableProvider>>> {
let table_ref = table_ref.into();
self.state
.read()
.schema_for_ref(table_ref)?
.register_table(table_ref.table().to_owned(), provider)
}
/// Deregisters the given table.
///
/// Returns the registered provider, if any
pub fn deregister_table<'a>(
&'a self,
table_ref: impl Into<TableReference<'a>>,
) -> Result<Option<Arc<dyn TableProvider>>> {
let table_ref = table_ref.into();
self.state
.read()
.schema_for_ref(table_ref)?
.deregister_table(table_ref.table())
}
/// Return true if the specified table exists in the schema provider.
pub fn table_exist<'a>(
&'a self,
table_ref: impl Into<TableReference<'a>>,
) -> Result<bool> {
let table_ref = table_ref.into();
Ok(self
.state
.read()
.schema_for_ref(table_ref)?
.table_exist(table_ref.table()))
}
/// Retrieves a [`DataFrame`] representing a table previously
/// registered by calling the [`register_table`] function.
///
/// Returns an error if no table has been registered with the
/// provided reference.
///
/// [`register_table`]: SessionContext::register_table
pub fn table<'a>(
&self,
table_ref: impl Into<TableReference<'a>>,
) -> Result<Arc<DataFrame>> {
let table_ref = table_ref.into();
let schema = self.state.read().schema_for_ref(table_ref)?;
match schema.table(table_ref.table()) {
Some(ref provider) => {
let plan = LogicalPlanBuilder::scan(
table_ref.table(),
provider_as_source(Arc::clone(provider)),
None,
)?
.build()?;
Ok(Arc::new(DataFrame::new(self.state.clone(), &plan)))
}
_ => Err(DataFusionError::Plan(format!(
"No table named '{}'",
table_ref.table()
))),
}
}
/// Returns the set of available tables in the default catalog and
/// schema.
///
/// Use [`table`] to get a specific table.
///
/// [`table`]: SessionContext::table
#[deprecated(
note = "Please use the catalog provider interface (`SessionContext::catalog`) to examine available catalogs, schemas, and tables"
)]
pub fn tables(&self) -> Result<HashSet<String>> {
Ok(self
.state
.read()
// a bare reference will always resolve to the default catalog and schema
.schema_for_ref(TableReference::Bare { table: "" })?
.table_names()
.iter()
.cloned()
.collect())
}
/// Optimizes the logical plan by applying optimizer rules.
pub fn optimize(&self, plan: &LogicalPlan) -> Result<LogicalPlan> {
self.state.read().optimize(plan)
}
/// Creates a physical plan from a logical plan.
pub async fn create_physical_plan(
&self,
logical_plan: &LogicalPlan,
) -> Result<Arc<dyn ExecutionPlan>> {
let state_cloned = {
let mut state = self.state.write();
state.execution_props.start_execution();
// We need to clone `state` to release the lock that is not `Send`. We could
// make the lock `Send` by using `tokio::sync::Mutex`, but that would require to
// propagate async even to the `LogicalPlan` building methods.
// Cloning `state` here is fine as we then pass it as immutable `&state`, which
// means that we avoid write consistency issues as the cloned version will not
// be written to. As for eventual modifications that would be applied to the
// original state after it has been cloned, they will not be picked up by the
// clone but that is okay, as it is equivalent to postponing the state update
// by keeping the lock until the end of the function scope.
state.clone()
};
state_cloned.create_physical_plan(logical_plan).await
}
/// Executes a query and writes the results to a partitioned CSV file.
pub async fn write_csv(
&self,
plan: Arc<dyn ExecutionPlan>,
path: impl AsRef<str>,
) -> Result<()> {
let state = self.state.read().clone();
plan_to_csv(&state, plan, path).await