@@ -60,8 +60,8 @@ use tokio::task::JoinHandle;
60
60
/// Configuration options for a SessionContext
61
61
#[ pyclass( name = "SessionConfig" , module = "datafusion" , subclass) ]
62
62
#[ derive( Clone , Default ) ]
63
- pub ( crate ) struct PySessionConfig {
64
- pub ( crate ) config : SessionConfig ,
63
+ pub struct PySessionConfig {
64
+ pub config : SessionConfig ,
65
65
}
66
66
67
67
impl From < SessionConfig > for PySessionConfig {
@@ -153,8 +153,8 @@ impl PySessionConfig {
153
153
/// Runtime options for a SessionContext
154
154
#[ pyclass( name = "RuntimeConfig" , module = "datafusion" , subclass) ]
155
155
#[ derive( Clone ) ]
156
- pub ( crate ) struct PyRuntimeConfig {
157
- pub ( crate ) config : RuntimeConfig ,
156
+ pub struct PyRuntimeConfig {
157
+ pub config : RuntimeConfig ,
158
158
}
159
159
160
160
#[ pymethods]
@@ -215,15 +215,18 @@ impl PyRuntimeConfig {
215
215
/// multi-threaded execution engine to perform the execution.
216
216
#[ pyclass( name = "SessionContext" , module = "datafusion" , subclass) ]
217
217
#[ derive( Clone ) ]
218
- pub ( crate ) struct PySessionContext {
219
- pub ( crate ) ctx : SessionContext ,
218
+ pub struct PySessionContext {
219
+ pub ctx : SessionContext ,
220
220
}
221
221
222
222
#[ pymethods]
223
223
impl PySessionContext {
224
224
#[ pyo3( signature = ( config=None , runtime=None ) ) ]
225
225
#[ new]
226
- fn new ( config : Option < PySessionConfig > , runtime : Option < PyRuntimeConfig > ) -> PyResult < Self > {
226
+ pub fn new (
227
+ config : Option < PySessionConfig > ,
228
+ runtime : Option < PyRuntimeConfig > ,
229
+ ) -> PyResult < Self > {
227
230
let config = if let Some ( c) = config {
228
231
c. config
229
232
} else {
@@ -242,7 +245,7 @@ impl PySessionContext {
242
245
}
243
246
244
247
/// Register a an object store with the given name
245
- fn register_object_store (
248
+ pub fn register_object_store (
246
249
& mut self ,
247
250
scheme : & str ,
248
251
store : & PyAny ,
@@ -276,13 +279,13 @@ impl PySessionContext {
276
279
}
277
280
278
281
/// Returns a PyDataFrame whose plan corresponds to the SQL statement.
279
- fn sql ( & mut self , query : & str , py : Python ) -> PyResult < PyDataFrame > {
282
+ pub fn sql ( & mut self , query : & str , py : Python ) -> PyResult < PyDataFrame > {
280
283
let result = self . ctx . sql ( query) ;
281
284
let df = wait_for_future ( py, result) . map_err ( DataFusionError :: from) ?;
282
285
Ok ( PyDataFrame :: new ( df) )
283
286
}
284
287
285
- fn create_dataframe (
288
+ pub fn create_dataframe (
286
289
& mut self ,
287
290
partitions : PyArrowType < Vec < Vec < RecordBatch > > > ,
288
291
name : Option < & str > ,
@@ -314,13 +317,13 @@ impl PySessionContext {
314
317
}
315
318
316
319
/// Create a DataFrame from an existing logical plan
317
- fn create_dataframe_from_logical_plan ( & mut self , plan : PyLogicalPlan ) -> PyDataFrame {
320
+ pub fn create_dataframe_from_logical_plan ( & mut self , plan : PyLogicalPlan ) -> PyDataFrame {
318
321
PyDataFrame :: new ( DataFrame :: new ( self . ctx . state ( ) , plan. plan . as_ref ( ) . clone ( ) ) )
319
322
}
320
323
321
324
/// Construct datafusion dataframe from Python list
322
325
#[ allow( clippy:: wrong_self_convention) ]
323
- fn from_pylist (
326
+ pub fn from_pylist (
324
327
& mut self ,
325
328
data : PyObject ,
326
329
name : Option < & str > ,
@@ -340,7 +343,7 @@ impl PySessionContext {
340
343
341
344
/// Construct datafusion dataframe from Python dictionary
342
345
#[ allow( clippy:: wrong_self_convention) ]
343
- fn from_pydict (
346
+ pub fn from_pydict (
344
347
& mut self ,
345
348
data : PyObject ,
346
349
name : Option < & str > ,
@@ -360,7 +363,7 @@ impl PySessionContext {
360
363
361
364
/// Construct datafusion dataframe from Arrow Table
362
365
#[ allow( clippy:: wrong_self_convention) ]
363
- fn from_arrow_table (
366
+ pub fn from_arrow_table (
364
367
& mut self ,
365
368
data : PyObject ,
366
369
name : Option < & str > ,
@@ -381,7 +384,7 @@ impl PySessionContext {
381
384
382
385
/// Construct datafusion dataframe from pandas
383
386
#[ allow( clippy:: wrong_self_convention) ]
384
- fn from_pandas (
387
+ pub fn from_pandas (
385
388
& mut self ,
386
389
data : PyObject ,
387
390
name : Option < & str > ,
@@ -401,7 +404,7 @@ impl PySessionContext {
401
404
402
405
/// Construct datafusion dataframe from polars
403
406
#[ allow( clippy:: wrong_self_convention) ]
404
- fn from_polars (
407
+ pub fn from_polars (
405
408
& mut self ,
406
409
data : PyObject ,
407
410
name : Option < & str > ,
@@ -417,21 +420,21 @@ impl PySessionContext {
417
420
} )
418
421
}
419
422
420
- fn register_table ( & mut self , name : & str , table : & PyTable ) -> PyResult < ( ) > {
423
+ pub fn register_table ( & mut self , name : & str , table : & PyTable ) -> PyResult < ( ) > {
421
424
self . ctx
422
425
. register_table ( name, table. table ( ) )
423
426
. map_err ( DataFusionError :: from) ?;
424
427
Ok ( ( ) )
425
428
}
426
429
427
- fn deregister_table ( & mut self , name : & str ) -> PyResult < ( ) > {
430
+ pub fn deregister_table ( & mut self , name : & str ) -> PyResult < ( ) > {
428
431
self . ctx
429
432
. deregister_table ( name)
430
433
. map_err ( DataFusionError :: from) ?;
431
434
Ok ( ( ) )
432
435
}
433
436
434
- fn register_record_batches (
437
+ pub fn register_record_batches (
435
438
& mut self ,
436
439
name : & str ,
437
440
partitions : PyArrowType < Vec < Vec < RecordBatch > > > ,
@@ -451,7 +454,7 @@ impl PySessionContext {
451
454
skip_metadata=true ,
452
455
schema=None ,
453
456
file_sort_order=None ) ) ]
454
- fn register_parquet (
457
+ pub fn register_parquet (
455
458
& mut self ,
456
459
name : & str ,
457
460
path : & str ,
@@ -489,7 +492,7 @@ impl PySessionContext {
489
492
schema_infer_max_records=1000 ,
490
493
file_extension=".csv" ,
491
494
file_compression_type=None ) ) ]
492
- fn register_csv (
495
+ pub fn register_csv (
493
496
& mut self ,
494
497
name : & str ,
495
498
path : PathBuf ,
@@ -533,7 +536,7 @@ impl PySessionContext {
533
536
file_extension=".json" ,
534
537
table_partition_cols=vec![ ] ,
535
538
file_compression_type=None ) ) ]
536
- fn register_json (
539
+ pub fn register_json (
537
540
& mut self ,
538
541
name : & str ,
539
542
path : PathBuf ,
@@ -568,7 +571,7 @@ impl PySessionContext {
568
571
file_extension=".avro" ,
569
572
table_partition_cols=vec![ ] ,
570
573
infinite=false ) ) ]
571
- fn register_avro (
574
+ pub fn register_avro (
572
575
& mut self ,
573
576
name : & str ,
574
577
path : PathBuf ,
@@ -595,7 +598,7 @@ impl PySessionContext {
595
598
}
596
599
597
600
// Registers a PyArrow.Dataset
598
- fn register_dataset ( & self , name : & str , dataset : & PyAny , py : Python ) -> PyResult < ( ) > {
601
+ pub fn register_dataset ( & self , name : & str , dataset : & PyAny , py : Python ) -> PyResult < ( ) > {
599
602
let table: Arc < dyn TableProvider > = Arc :: new ( Dataset :: new ( dataset, py) ?) ;
600
603
601
604
self . ctx
@@ -605,18 +608,18 @@ impl PySessionContext {
605
608
Ok ( ( ) )
606
609
}
607
610
608
- fn register_udf ( & mut self , udf : PyScalarUDF ) -> PyResult < ( ) > {
611
+ pub fn register_udf ( & mut self , udf : PyScalarUDF ) -> PyResult < ( ) > {
609
612
self . ctx . register_udf ( udf. function ) ;
610
613
Ok ( ( ) )
611
614
}
612
615
613
- fn register_udaf ( & mut self , udaf : PyAggregateUDF ) -> PyResult < ( ) > {
616
+ pub fn register_udaf ( & mut self , udaf : PyAggregateUDF ) -> PyResult < ( ) > {
614
617
self . ctx . register_udaf ( udaf. function ) ;
615
618
Ok ( ( ) )
616
619
}
617
620
618
621
#[ pyo3( signature = ( name="datafusion" ) ) ]
619
- fn catalog ( & self , name : & str ) -> PyResult < PyCatalog > {
622
+ pub fn catalog ( & self , name : & str ) -> PyResult < PyCatalog > {
620
623
match self . ctx . catalog ( name) {
621
624
Some ( catalog) => Ok ( PyCatalog :: new ( catalog) ) ,
622
625
None => Err ( PyKeyError :: new_err ( format ! (
@@ -626,31 +629,31 @@ impl PySessionContext {
626
629
}
627
630
}
628
631
629
- fn tables ( & self ) -> HashSet < String > {
632
+ pub fn tables ( & self ) -> HashSet < String > {
630
633
#[ allow( deprecated) ]
631
634
self . ctx . tables ( ) . unwrap ( )
632
635
}
633
636
634
- fn table ( & self , name : & str , py : Python ) -> PyResult < PyDataFrame > {
637
+ pub fn table ( & self , name : & str , py : Python ) -> PyResult < PyDataFrame > {
635
638
let x = wait_for_future ( py, self . ctx . table ( name) ) . map_err ( DataFusionError :: from) ?;
636
639
Ok ( PyDataFrame :: new ( x) )
637
640
}
638
641
639
- fn table_exist ( & self , name : & str ) -> PyResult < bool > {
642
+ pub fn table_exist ( & self , name : & str ) -> PyResult < bool > {
640
643
Ok ( self . ctx . table_exist ( name) ?)
641
644
}
642
645
643
- fn empty_table ( & self ) -> PyResult < PyDataFrame > {
646
+ pub fn empty_table ( & self ) -> PyResult < PyDataFrame > {
644
647
Ok ( PyDataFrame :: new ( self . ctx . read_empty ( ) ?) )
645
648
}
646
649
647
- fn session_id ( & self ) -> String {
650
+ pub fn session_id ( & self ) -> String {
648
651
self . ctx . session_id ( )
649
652
}
650
653
651
654
#[ allow( clippy:: too_many_arguments) ]
652
655
#[ pyo3( signature = ( path, schema=None , schema_infer_max_records=1000 , file_extension=".json" , table_partition_cols=vec![ ] , file_compression_type=None ) ) ]
653
- fn read_json (
656
+ pub fn read_json (
654
657
& mut self ,
655
658
path : PathBuf ,
656
659
schema : Option < PyArrowType < Schema > > ,
@@ -689,7 +692,7 @@ impl PySessionContext {
689
692
file_extension=".csv" ,
690
693
table_partition_cols=vec![ ] ,
691
694
file_compression_type=None ) ) ]
692
- fn read_csv (
695
+ pub fn read_csv (
693
696
& self ,
694
697
path : PathBuf ,
695
698
schema : Option < PyArrowType < Schema > > ,
@@ -741,7 +744,7 @@ impl PySessionContext {
741
744
skip_metadata=true ,
742
745
schema=None ,
743
746
file_sort_order=None ) ) ]
744
- fn read_parquet (
747
+ pub fn read_parquet (
745
748
& self ,
746
749
path : & str ,
747
750
table_partition_cols : Vec < ( String , String ) > ,
@@ -771,7 +774,7 @@ impl PySessionContext {
771
774
772
775
#[ allow( clippy:: too_many_arguments) ]
773
776
#[ pyo3( signature = ( path, schema=None , table_partition_cols=vec![ ] , file_extension=".avro" ) ) ]
774
- fn read_avro (
777
+ pub fn read_avro (
775
778
& self ,
776
779
path : & str ,
777
780
schema : Option < PyArrowType < Schema > > ,
@@ -793,7 +796,7 @@ impl PySessionContext {
793
796
Ok ( PyDataFrame :: new ( df) )
794
797
}
795
798
796
- fn read_table ( & self , table : & PyTable ) -> PyResult < PyDataFrame > {
799
+ pub fn read_table ( & self , table : & PyTable ) -> PyResult < PyDataFrame > {
797
800
let df = self
798
801
. ctx
799
802
. read_table ( table. table ( ) )
0 commit comments