-
Notifications
You must be signed in to change notification settings - Fork 112
/
dataframe.py
1979 lines (1632 loc) · 79.4 KB
/
dataframe.py
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
# isort: dont-add-import: from __future__ import annotations
#
# This file uses strings for forward type annotations in public APIs,
# in order to support runtime typechecking across different Python versions.
# For technical details, see https://github.com/Eventual-Inc/Daft/pull/630
import os
import pathlib
import warnings
from dataclasses import dataclass
from functools import reduce
from typing import (
TYPE_CHECKING,
Any,
Callable,
Dict,
Iterable,
Iterator,
List,
Optional,
Set,
Tuple,
TypeVar,
Union,
)
from daft.api_annotations import DataframePublicAPI
from daft.context import get_context
from daft.convert import InputListType
from daft.daft import FileFormat, IOConfig, JoinStrategy, JoinType, ResourceRequest
from daft.dataframe.preview import DataFramePreview
from daft.datatype import DataType
from daft.errors import ExpressionTypeError
from daft.expressions import Expression, ExpressionsProjection, col, lit
from daft.logical.builder import LogicalPlanBuilder
from daft.runners.partitioning import PartitionCacheEntry, PartitionSet
from daft.runners.pyrunner import LocalPartitionSet
from daft.table import MicroPartition
from daft.viz import DataFrameDisplay
if TYPE_CHECKING:
import dask
import pandas as pd
import pyarrow as pa
import torch.utils.data.Dataset as TorchDataset
import torch.utils.data.IterableDataset as TorchIterableDataset
from pyiceberg.table import Table as IcebergTable
from ray import ObjectRef as RayObjectRef
from ray.data.dataset import Dataset as RayDataset
from daft.logical.schema import Schema
UDFReturnType = TypeVar("UDFReturnType", covariant=True)
ColumnInputType = Union[Expression, str]
ManyColumnsInputType = Union[ColumnInputType, Iterable[ColumnInputType]]
class DataFrame:
"""A Daft DataFrame is a table of data. It has columns, where each column has a type and the same
number of items (rows) as all other columns.
"""
def __init__(self, builder: LogicalPlanBuilder) -> None:
"""Constructs a DataFrame according to a given LogicalPlan. Users are expected instead to call
the classmethods on DataFrame to create a DataFrame.
Args:
plan: LogicalPlan describing the steps required to arrive at this DataFrame
"""
if not isinstance(builder, LogicalPlanBuilder):
if isinstance(builder, dict):
raise ValueError(
"DataFrames should be constructed with a dictionary of columns using `daft.from_pydict`"
)
if isinstance(builder, list):
raise ValueError(
"DataFrames should be constructed with a list of dictionaries using `daft.from_pylist`"
)
raise ValueError(f"Expected DataFrame to be constructed with a LogicalPlanBuilder, received: {builder}")
self.__builder = builder
self._result_cache: Optional[PartitionCacheEntry] = None
self._preview = DataFramePreview(preview_partition=None, dataframe_num_rows=None)
self._num_preview_rows = get_context().daft_execution_config.num_preview_rows
@property
def _builder(self) -> LogicalPlanBuilder:
if self._result_cache is None:
return self.__builder
else:
num_partitions = self._result_cache.num_partitions()
size_bytes = self._result_cache.size_bytes()
num_rows = self._result_cache.num_rows()
# Partition set should always be set on cache entry.
assert (
num_partitions is not None and size_bytes is not None and num_rows is not None
), "Partition set should always be set on cache entry"
return self.__builder.from_in_memory_scan(
self._result_cache,
self.__builder.schema(),
num_partitions=num_partitions,
size_bytes=size_bytes,
num_rows=num_rows,
)
def _get_current_builder(self) -> LogicalPlanBuilder:
"""Returns the current logical plan builder, without any caching optimizations."""
return self.__builder
@property
def _result(self) -> Optional[PartitionSet]:
if self._result_cache is None:
return None
else:
return self._result_cache.value
@DataframePublicAPI
def explain(self, show_all: bool = False, simple: bool = False) -> None:
"""Prints the (logical and physical) plans that will be executed to produce this DataFrame.
Defaults to showing the unoptimized logical plan. Use ``show_all=True`` to show the unoptimized logical plan,
the optimized logical plan, and the physical plan.
Args:
show_all (bool): Whether to show the optimized logical plan and the physical plan in addition to the
unoptimized logical plan.
simple (bool): Whether to only show the type of op for each node in the plan, rather than showing details
of how each op is configured.
"""
if self._result_cache is not None:
print("Result is cached and will skip computation\n")
print(self._builder.pretty_print(simple))
print("However here is the logical plan used to produce this result:\n")
builder = self.__builder
print("== Unoptimized Logical Plan ==\n")
print(builder.pretty_print(simple))
if show_all:
print("\n== Optimized Logical Plan ==\n")
builder = builder.optimize()
print(builder.pretty_print(simple))
print("\n== Physical Plan ==\n")
physical_plan_scheduler = builder.to_physical_plan_scheduler(get_context().daft_execution_config)
print(physical_plan_scheduler.pretty_print(simple))
else:
print(
"\n \nSet `show_all=True` to also see the Optimized and Physical plans. This will run the query optimizer."
)
def num_partitions(self) -> int:
daft_execution_config = get_context().daft_execution_config
# We need to run the optimizer since that could change the number of partitions
return self.__builder.optimize().to_physical_plan_scheduler(daft_execution_config).num_partitions()
@DataframePublicAPI
def schema(self) -> Schema:
"""Returns the Schema of the DataFrame, which provides information about each column
Returns:
Schema: schema of the DataFrame
"""
return self.__builder.schema()
@property
def column_names(self) -> List[str]:
"""Returns column names of DataFrame as a list of strings.
Returns:
List[str]: Column names of this DataFrame.
"""
return self.__builder.schema().column_names()
@property
def columns(self) -> List[Expression]:
"""Returns column of DataFrame as a list of Expressions.
Returns:
List[Expression]: Columns of this DataFrame.
"""
return [col(field.name) for field in self.__builder.schema()]
@DataframePublicAPI
def __iter__(self) -> Iterator[Dict[str, Any]]:
"""Return an iterator of rows for this dataframe.
Each row will be a pydict of the form { "key" : value }.
"""
if self._result is not None:
# If the dataframe has already finished executing,
# use the precomputed results.
pydict = self.to_pydict()
for i in range(len(self)):
row = {key: value[i] for (key, value) in pydict.items()}
yield row
else:
# Execute the dataframe in a streaming fashion.
context = get_context()
partitions_iter = context.runner().run_iter_tables(self._builder)
# Iterate through partitions.
for partition in partitions_iter:
pydict = partition.to_pydict()
# Yield invidiual rows from the partition.
for i in range(len(partition)):
row = {key: value[i] for (key, value) in pydict.items()}
yield row
@DataframePublicAPI
def iter_partitions(self) -> Iterator[Union[MicroPartition, "RayObjectRef"]]:
"""Begin executing this dataframe and return an iterator over the partitions.
Each partition will be returned as a daft.Table object (if using Python runner backend)
or a ray ObjectRef (if using Ray runner backend).
"""
if self._result is not None:
# If the dataframe has already finished executing,
# use the precomputed results.
for mat_result in self._result.values():
yield mat_result.partition()
else:
# Execute the dataframe in a streaming fashion.
context = get_context()
results_iter = context.runner().run_iter(self._builder)
for result in results_iter:
yield result.partition()
def _populate_preview(self) -> None:
"""Populates the preview of the DataFrame, if it is not already populated."""
if self._result is None:
return
preview_partition_invalid = (
self._preview.preview_partition is None or len(self._preview.preview_partition) < self._num_preview_rows
)
if preview_partition_invalid:
preview_parts = self._result._get_preview_vpartition(self._num_preview_rows)
preview_results = LocalPartitionSet()
for i, part in enumerate(preview_parts):
preview_results.set_partition_from_table(i, part)
preview_partition = preview_results._get_merged_vpartition()
self._preview = DataFramePreview(
preview_partition=preview_partition,
dataframe_num_rows=len(self),
)
@DataframePublicAPI
def __repr__(self) -> str:
self._populate_preview()
display = DataFrameDisplay(self._preview, self.schema())
return display.__repr__()
@DataframePublicAPI
def _repr_html_(self) -> str:
self._populate_preview()
display = DataFrameDisplay(self._preview, self.schema())
return display._repr_html_()
###
# Creation methods
###
@classmethod
def _from_pylist(cls, data: List[Dict[str, Any]]) -> "DataFrame":
"""Creates a DataFrame from a list of dictionaries."""
headers: Set[str] = set()
for row in data:
if not isinstance(row, dict):
raise ValueError(f"Expected list of dictionaries of {{column_name: value}}, received: {type(row)}")
headers.update(row.keys())
headers_ordered = sorted(list(headers))
return cls._from_pydict(data={header: [row.get(header, None) for row in data] for header in headers_ordered})
@classmethod
def _from_pydict(cls, data: Dict[str, InputListType]) -> "DataFrame":
"""Creates a DataFrame from a Python dictionary."""
column_lengths = {key: len(data[key]) for key in data}
if len(set(column_lengths.values())) > 1:
raise ValueError(
f"Expected all columns to be of the same length, but received columns with lengths: {column_lengths}"
)
data_vpartition = MicroPartition.from_pydict(data)
return cls._from_tables(data_vpartition)
@classmethod
def _from_arrow(cls, data: Union["pa.Table", List["pa.Table"]]) -> "DataFrame":
"""Creates a DataFrame from a pyarrow Table."""
if not isinstance(data, list):
data = [data]
data_vpartitions = [MicroPartition.from_arrow(table) for table in data]
return cls._from_tables(*data_vpartitions)
@classmethod
def _from_pandas(cls, data: Union["pd.DataFrame", List["pd.DataFrame"]]) -> "DataFrame":
"""Creates a Daft DataFrame from a pandas DataFrame."""
if not isinstance(data, list):
data = [data]
data_vpartitions = [MicroPartition.from_pandas(df) for df in data]
return cls._from_tables(*data_vpartitions)
@classmethod
def _from_tables(cls, *parts: MicroPartition) -> "DataFrame":
"""Creates a Daft DataFrame from a single Table.
Args:
parts: The Tables that we wish to convert into a Daft DataFrame.
Returns:
DataFrame: Daft DataFrame created from the provided Table.
"""
if not parts:
raise ValueError("Can't create a DataFrame from an empty list of tables.")
result_pset = LocalPartitionSet()
for i, part in enumerate(parts):
result_pset.set_partition_from_table(i, part)
context = get_context()
cache_entry = context.runner().put_partition_set_into_cache(result_pset)
size_bytes = result_pset.size_bytes()
num_rows = len(result_pset)
assert size_bytes is not None, "In-memory data should always have non-None size in bytes"
builder = LogicalPlanBuilder.from_in_memory_scan(
cache_entry, parts[0].schema(), result_pset.num_partitions(), size_bytes, num_rows=num_rows
)
df = cls(builder)
df._result_cache = cache_entry
# build preview
df._populate_preview()
return df
###
# Write methods
###
@DataframePublicAPI
def write_parquet(
self,
root_dir: Union[str, pathlib.Path],
compression: str = "snappy",
partition_cols: Optional[List[ColumnInputType]] = None,
io_config: Optional[IOConfig] = None,
) -> "DataFrame":
"""Writes the DataFrame as parquet files, returning a new DataFrame with paths to the files that were written
Files will be written to ``<root_dir>/*`` with randomly generated UUIDs as the file names.
.. NOTE::
This call is **blocking** and will execute the DataFrame when called
Args:
root_dir (str): root file path to write parquet files to.
compression (str, optional): compression algorithm. Defaults to "snappy".
partition_cols (Optional[List[ColumnInputType]], optional): How to subpartition each partition further. Defaults to None.
io_config (Optional[IOConfig], optional): configurations to use when interacting with remote storage.
Returns:
DataFrame: The filenames that were written out as strings.
.. NOTE::
This call is **blocking** and will execute the DataFrame when called
"""
io_config = get_context().daft_planning_config.default_io_config if io_config is None else io_config
cols: Optional[List[Expression]] = None
if partition_cols is not None:
cols = self.__column_input_to_expression(tuple(partition_cols))
builder = self._builder.write_tabular(
root_dir=root_dir,
partition_cols=cols,
file_format=FileFormat.Parquet,
compression=compression,
io_config=io_config,
)
# Block and write, then retrieve data
write_df = DataFrame(builder)
write_df.collect()
assert write_df._result is not None
# Populate and return a new disconnected DataFrame
result_df = DataFrame(write_df._builder)
result_df._result_cache = write_df._result_cache
result_df._preview = write_df._preview
return result_df
@DataframePublicAPI
def write_csv(
self,
root_dir: Union[str, pathlib.Path],
partition_cols: Optional[List[ColumnInputType]] = None,
io_config: Optional[IOConfig] = None,
) -> "DataFrame":
"""Writes the DataFrame as CSV files, returning a new DataFrame with paths to the files that were written
Files will be written to ``<root_dir>/*`` with randomly generated UUIDs as the file names.
.. NOTE::
This call is **blocking** and will execute the DataFrame when called
Args:
root_dir (str): root file path to write parquet files to.
partition_cols (Optional[List[ColumnInputType]], optional): How to subpartition each partition further. Defaults to None.
io_config (Optional[IOConfig], optional): configurations to use when interacting with remote storage.
Returns:
DataFrame: The filenames that were written out as strings.
"""
io_config = get_context().daft_planning_config.default_io_config if io_config is None else io_config
cols: Optional[List[Expression]] = None
if partition_cols is not None:
cols = self.__column_input_to_expression(tuple(partition_cols))
builder = self._builder.write_tabular(
root_dir=root_dir,
partition_cols=cols,
file_format=FileFormat.Csv,
io_config=io_config,
)
# Block and write, then retrieve data
write_df = DataFrame(builder)
write_df.collect()
assert write_df._result is not None
# Populate and return a new disconnected DataFrame
result_df = DataFrame(write_df._builder)
result_df._result_cache = write_df._result_cache
result_df._preview = write_df._preview
return result_df
@DataframePublicAPI
def write_iceberg(self, table: "IcebergTable", mode: str = "append") -> "DataFrame":
"""Writes the DataFrame to an Iceberg Table, returning a new DataFrame with the operations that occurred.
Can be run in either `append` or `overwrite` mode which will either appends the rows in the DataFrame or will delete the existing rows and then append the DataFrame rows respectively.
.. NOTE::
This call is **blocking** and will execute the DataFrame when called
Args:
table (IcebergTable): Destination Iceberg Table to write dataframe to.
mode (str, optional): Operation mode of the write. `append` or `overwrite` Iceberg Table. Defaults to "append".
Returns:
DataFrame: The operations that occurred with this write.
"""
if len(table.spec().fields) > 0:
raise ValueError("Cannot write to partitioned Iceberg tables")
import pyarrow as pa
import pyiceberg
from packaging.version import parse
if parse(pyiceberg.__version__) < parse("0.6.0"):
raise ValueError(f"Write Iceberg is only supported on pyiceberg>=0.6.0, found {pyiceberg.__version__}")
if parse(pa.__version__) < parse("8.0.0"):
raise ValueError(f"Write Iceberg is only supported on pyarrow>=8.0.0, found {pa.__version__}")
from pyiceberg.table import _MergingSnapshotProducer
from pyiceberg.table.snapshots import Operation
operations = []
path = []
rows = []
size = []
if mode == "append":
operation = Operation.APPEND
elif mode == "overwrite":
operation = Operation.OVERWRITE
else:
raise ValueError(f"Only support `append` or `overwrite` mode. {mode} is unsupported")
# We perform the merge here since IcebergTable is not pickle-able
# We should be able to move to a transaction API for iceberg 0.7.0
merge = _MergingSnapshotProducer(operation=operation, table=table)
builder = self._builder.write_iceberg(table)
write_df = DataFrame(builder)
write_df.collect()
write_result = write_df.to_pydict()
assert "data_file" in write_result
data_files = write_result["data_file"]
if operation == Operation.OVERWRITE:
deleted_files = table.scan().plan_files()
else:
deleted_files = []
for data_file in data_files:
merge.append_data_file(data_file)
operations.append("ADD")
path.append(data_file.file_path)
rows.append(data_file.record_count)
size.append(data_file.file_size_in_bytes)
for pf in deleted_files:
data_file = pf.file
operations.append("DELETE")
path.append(data_file.file_path)
rows.append(data_file.record_count)
size.append(data_file.file_size_in_bytes)
merge.commit()
from daft import from_pydict
with_operations = from_pydict(
{
"operation": pa.array(operations, type=pa.string()),
"rows": pa.array(rows, type=pa.int64()),
"file_size": pa.array(size, type=pa.int64()),
"file_name": pa.array([os.path.basename(fp) for fp in path], type=pa.string()),
}
)
# NOTE: We are losing the history of the plan here.
# This is due to the fact that the logical plan of the write_iceberg returns datafiles but we want to return the above data
return with_operations
###
# DataFrame operations
###
def __column_input_to_expression(self, columns: Iterable[ColumnInputType]) -> List[Expression]:
# TODO(Kevin): remove this method and use _column_inputs_to_expressions
return [col(c) if isinstance(c, str) else c for c in columns]
def _is_column_input(self, x: Any) -> bool:
return isinstance(x, str) or isinstance(x, Expression)
def _column_inputs_to_expressions(self, columns: ManyColumnsInputType) -> List[Expression]:
"""
Inputs to dataframe operations can be passed in as individual arguments or an iterable.
In addition, they may be strings or Expressions.
This method normalizes the inputs to a list of Expressions.
"""
column_iter: Iterable[ColumnInputType] = [columns] if self._is_column_input(columns) else columns # type: ignore
return [col(c) if isinstance(c, str) else c for c in column_iter]
def _wildcard_inputs_to_expressions(self, columns: Tuple[ManyColumnsInputType, ...]) -> List[Expression]:
"""Handles wildcard argument column inputs"""
column_input: Iterable[ColumnInputType] = columns[0] if len(columns) == 1 else columns # type: ignore
return self._column_inputs_to_expressions(column_input)
def __getitem__(self, item: Union[slice, int, str, Iterable[Union[str, int]]]) -> Union[Expression, "DataFrame"]:
"""Gets a column from the DataFrame as an Expression (``df["mycol"]``)"""
result: Optional[Expression]
if isinstance(item, int):
schema = self._builder.schema()
if item < -len(schema) or item >= len(schema):
raise ValueError(f"{item} out of bounds for {schema}")
result = ExpressionsProjection.from_schema(schema)[item]
assert result is not None
return result
elif isinstance(item, str):
schema = self._builder.schema()
field = schema[item]
return col(field.name)
elif isinstance(item, Iterable):
schema = self._builder.schema()
columns = []
for it in item:
if isinstance(it, str):
result = col(schema[it].name)
columns.append(result)
elif isinstance(it, int):
if it < -len(schema) or it >= len(schema):
raise ValueError(f"{it} out of bounds for {schema}")
field = list(self._builder.schema())[it]
columns.append(col(field.name))
else:
raise ValueError(f"unknown indexing type: {type(it)}")
return self.select(*columns)
elif isinstance(item, slice):
schema = self._builder.schema()
columns_exprs: ExpressionsProjection = ExpressionsProjection.from_schema(schema)
selected_columns = columns_exprs[item]
return self.select(*selected_columns)
else:
raise ValueError(f"unknown indexing type: {type(item)}")
def _add_monotonically_increasing_id(self, column_name: Optional[str] = None) -> "DataFrame":
"""Generates a column of monotonically increasing unique ids for the DataFrame.
The implementation of this method puts the partition number in the upper 28 bits, and the row number in each partition
in the lower 36 bits. This allows for 2^28 ≈ 268 million partitions and 2^40 ≈ 68 billion rows per partition.
Example:
>>> df = daft.from_pydict({"a": [1, 2, 3, 4]}).into_partitions(2)
>>> df = df._add_monotonically_increasing_id()
>>> df.show()
╭─────────────┬───────╮
│ id ┆ a │
│ --- ┆ --- │
│ UInt64 ┆ Int64 │
╞═════════════╪═══════╡
│ 0 ┆ 1 │
├╌╌╌╌╌╌╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┤
│ 1 ┆ 2 │
├╌╌╌╌╌╌╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┤
│ 68719476736 ┆ 3 │
├╌╌╌╌╌╌╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┤
│ 68719476737 ┆ 4 │
╰─────────────┴───────╯
(Showing first 4 of 4 rows)
Args:
column_name (Optional[str], optional): name of the new column. Defaults to "id".
Returns:
DataFrame: DataFrame with a new column of monotonically increasing ids.
"""
builder = self._builder.add_monotonically_increasing_id(column_name)
return DataFrame(builder)
@DataframePublicAPI
def select(self, *columns: ColumnInputType) -> "DataFrame":
"""Creates a new DataFrame from the provided expressions, similar to a SQL ``SELECT``
Example:
>>> # names of columns as strings
>>> df = df.select('x', 'y')
>>>
>>> # names of columns as expressions
>>> df = df.select(col('x'), col('y'))
>>>
>>> # call expressions
>>> df = df.select(col('x') * col('y'))
>>>
>>> # any mix of the above
>>> df = df.select('x', col('y'), col('z') + 1)
Args:
*columns (Union[str, Expression]): columns to select from the current DataFrame
Returns:
DataFrame: new DataFrame that will select the passed in columns
"""
assert len(columns) > 0
builder = self._builder.select(self.__column_input_to_expression(columns))
return DataFrame(builder)
@DataframePublicAPI
def distinct(self) -> "DataFrame":
"""Computes unique rows, dropping duplicates
Example:
>>> unique_df = df.distinct()
Returns:
DataFrame: DataFrame that has only unique rows.
"""
ExpressionsProjection.from_schema(self._builder.schema())
builder = self._builder.distinct()
return DataFrame(builder)
@DataframePublicAPI
def sample(
self,
fraction: float,
with_replacement: bool = False,
seed: Optional[int] = None,
) -> "DataFrame":
"""Samples a fraction of rows from the DataFrame
Example:
>>> sampled_df = df.sample(0.5)
Args:
fraction (float): fraction of rows to sample.
with_replacement (bool, optional): whether to sample with replacement. Defaults to False.
seed (Optional[int], optional): random seed. Defaults to None.
Returns:
DataFrame: DataFrame with a fraction of rows.
"""
if fraction < 0.0 or fraction > 1.0:
raise ValueError(f"fraction should be between 0.0 and 1.0, but got {fraction}")
builder = self._builder.sample(fraction, with_replacement, seed)
return DataFrame(builder)
@DataframePublicAPI
def exclude(self, *names: str) -> "DataFrame":
"""Drops columns from the current DataFrame by name
This is equivalent of performing a select with all the columns but the ones excluded.
Example:
>>> df_without_x = df.exclude('x')
Args:
*names (str): names to exclude
Returns:
DataFrame: DataFrame with some columns excluded.
"""
builder = self._builder.exclude(list(names))
return DataFrame(builder)
@DataframePublicAPI
def where(self, predicate: Expression) -> "DataFrame":
"""Filters rows via a predicate expression, similar to SQL ``WHERE``.
Example:
>>> filtered_df = df.where((col('x') < 10) & (col('y') == 10))
Args:
predicate (Expression): expression that keeps row if evaluates to True.
Returns:
DataFrame: Filtered DataFrame.
"""
builder = self._builder.filter(predicate)
return DataFrame(builder)
@DataframePublicAPI
def with_column(
self,
column_name: str,
expr: Expression,
resource_request: ResourceRequest = ResourceRequest(),
) -> "DataFrame":
"""Adds a column to the current DataFrame with an Expression, equivalent to a ``select``
with all current columns and the new one
Example:
>>> new_df = df.with_column('x+1', col('x') + 1)
Args:
column_name (str): name of new column
expr (Expression): expression of the new column.
resource_request (ResourceRequest): a custom resource request for the execution of this operation
Returns:
DataFrame: DataFrame with new column.
"""
return self.with_columns({column_name: expr}, resource_request)
@DataframePublicAPI
def with_columns(
self,
columns: Dict[str, Expression],
resource_request: ResourceRequest = ResourceRequest(),
) -> "DataFrame":
"""Adds columns to the current DataFrame with Expressions, equivalent to a ``select``
with all current columns and the new ones
Example:
>>> df = daft.from_pydict({'x': [1, 2, 3], 'y': [4, 5, 6]})
>>>
>>> new_df = df.with_columns({
'foo': df['x'] + 1,
'bar': df['y'] - df['x']
})
>>> new_df.show()
╭───────┬───────┬───────┬───────╮
│ x ┆ y ┆ foo ┆ bar │
│ --- ┆ --- ┆ --- ┆ --- │
│ Int64 ┆ Int64 ┆ Int64 ┆ Int64 │
╞═══════╪═══════╪═══════╪═══════╡
│ 1 ┆ 4 ┆ 2 ┆ 3 │
├╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┤
│ 2 ┆ 5 ┆ 3 ┆ 3 │
├╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┼╌╌╌╌╌╌╌┤
│ 3 ┆ 6 ┆ 4 ┆ 3 │
╰───────┴───────┴───────┴───────╯
(Showing first 3 of 3 rows)
Args:
columns (Dict[str, Expression]): Dictionary of new columns in the format { name: expression }
resource_request (ResourceRequest): a custom resource request for the execution of this operation
Returns:
DataFrame: DataFrame with new columns.
"""
if not isinstance(resource_request, ResourceRequest):
raise TypeError(f"resource_request should be a ResourceRequest, but got {type(resource_request)}")
new_columns = [col.alias(name) for name, col in columns.items()]
builder = self._builder.with_columns(new_columns, resource_request)
return DataFrame(builder)
@DataframePublicAPI
def sort(
self,
by: Union[ColumnInputType, List[ColumnInputType]],
desc: Union[bool, List[bool]] = False,
) -> "DataFrame":
"""Sorts DataFrame globally
Example:
>>> sorted_df = df.sort(col('x') + col('y'))
>>> sorted_df = df.sort([col('x'), col('y')], desc=[False, True])
>>> sorted_df = df.sort(['z', col('x'), col('y')], desc=[True, False, True])
Note:
* Since this a global sort, this requires an expensive repartition which can be quite slow.
* Supports multicolumn sorts and can have unique `descending` flag per column.
Args:
column (Union[ColumnInputType, List[ColumnInputType]]): column to sort by. Can be `str` or expression as well as a list of either.
desc (Union[bool, List[bool]), optional): Sort by descending order. Defaults to False.
Returns:
DataFrame: Sorted DataFrame.
"""
if not isinstance(by, list):
by = [
by,
]
sort_by = self.__column_input_to_expression(by)
builder = self._builder.sort(sort_by=sort_by, descending=desc)
return DataFrame(builder)
@DataframePublicAPI
def limit(self, num: int) -> "DataFrame":
"""Limits the rows in the DataFrame to the first ``N`` rows, similar to a SQL ``LIMIT``
Example:
>>> df_limited = df.limit(10) # returns 10 rows
Args:
num (int): maximum rows to allow.
eager (bool): whether to maximize for latency (time to first result) by eagerly executing
only one partition at a time, or throughput by executing multiple limits at a time
Returns:
DataFrame: Limited DataFrame
"""
builder = self._builder.limit(num, eager=False)
return DataFrame(builder)
@DataframePublicAPI
def count_rows(self) -> int:
"""Executes the Dataframe to count the number of rows.
Returns:
int: count of the number of rows in this DataFrame.
"""
builder = self._builder.count()
count_df = DataFrame(builder)
# Expects builder to produce a single-partition, single-row DataFrame containing
# a "count" column, where the lone value represents the row count for the DataFrame.
return count_df.to_pydict()["count"][0]
@DataframePublicAPI
def repartition(self, num: Optional[int], *partition_by: ColumnInputType) -> "DataFrame":
"""Repartitions DataFrame to ``num`` partitions
If columns are passed in, then DataFrame will be repartitioned by those, otherwise
random repartitioning will occur.
.. NOTE::
This function will globally shuffle your data, which is potentially a very expensive operation.
If instead you merely wish to "split" or "coalesce" partitions to obtain a target number of partitions,
you mean instead wish to consider using :meth:`DataFrame.into_partitions <daft.DataFrame.into_partitions>`
which avoids shuffling of data in favor of splitting/coalescing adjacent partitions where appropriate.
Example:
>>> random_repart_df = df.repartition(4)
>>> part_by_df = df.repartition(4, 'x', col('y') + 1)
Args:
num (Optional[int]): Number of target partitions; if None, the number of partitions will not be changed.
*partition_by (Union[str, Expression]): Optional columns to partition by.
Returns:
DataFrame: Repartitioned DataFrame.
"""
if len(partition_by) == 0:
warnings.warn(
"No columns specified for repartition, so doing a random shuffle. If you do not require rebalancing of "
"partitions, you may instead prefer using `df.into_partitions(N)` which is a cheaper operation that "
"avoids shuffling data."
)
builder = self._builder.random_shuffle(num)
else:
builder = self._builder.hash_repartition(num, self.__column_input_to_expression(partition_by))
return DataFrame(builder)
@DataframePublicAPI
def into_partitions(self, num: int) -> "DataFrame":
"""Splits or coalesces DataFrame to ``num`` partitions. Order is preserved.
No rebalancing is done; the minimum number of splits or merges are applied.
(i.e. if there are 2 partitions, and change it into 3, this function will just split the bigger one)
Example:
>>> df_with_5_partitions = df.into_partitions(5)
Args:
num (int): number of target partitions.
Returns:
DataFrame: Dataframe with ``num`` partitions.
"""
builder = self._builder.into_partitions(num)
return DataFrame(builder)
@DataframePublicAPI
def join(
self,
other: "DataFrame",
on: Optional[Union[List[ColumnInputType], ColumnInputType]] = None,
left_on: Optional[Union[List[ColumnInputType], ColumnInputType]] = None,
right_on: Optional[Union[List[ColumnInputType], ColumnInputType]] = None,
how: str = "inner",
strategy: Optional[str] = None,
) -> "DataFrame":
"""Column-wise join of the current DataFrame with an ``other`` DataFrame, similar to a SQL ``JOIN``
.. NOTE::
Although self joins are supported, we currently duplicate the logical plan for the right side
and recompute the entire tree. Caching for this is on the roadmap.
Args:
other (DataFrame): the right DataFrame to join on.
on (Optional[Union[List[ColumnInputType], ColumnInputType]], optional): key or keys to join on [use if the keys on the left and right side match.]. Defaults to None.
left_on (Optional[Union[List[ColumnInputType], ColumnInputType]], optional): key or keys to join on left DataFrame. Defaults to None.
right_on (Optional[Union[List[ColumnInputType], ColumnInputType]], optional): key or keys to join on right DataFrame. Defaults to None.
how (str, optional): what type of join to perform; currently "inner", "left", "right", and "outer" are supported. Defaults to "inner".
strategy (Optional[str]): The join strategy (algorithm) to use; currently "hash", "sort_merge", "broadcast", and None are supported, where None
chooses the join strategy automatically during query optimization. The default is None.
Raises:
ValueError: if `on` is passed in and `left_on` or `right_on` is not None.
ValueError: if `on` is None but both `left_on` and `right_on` are not defined.
Returns:
DataFrame: Joined DataFrame.
"""
if on is None:
if left_on is None or right_on is None:
raise ValueError("If `on` is None then both `left_on` and `right_on` must not be None")
else:
if left_on is not None or right_on is not None:
raise ValueError("If `on` is not None then both `left_on` and `right_on` must be None")
left_on = on
right_on = on
join_type = JoinType.from_join_type_str(how)
join_strategy = JoinStrategy.from_join_strategy_str(strategy) if strategy is not None else None
if join_strategy == JoinStrategy.SortMerge and join_type != JoinType.Inner:
raise ValueError("Sort merge join only supports inner joins")
if join_strategy == JoinStrategy.Broadcast and join_type == JoinType.Outer:
raise ValueError("Broadcast join does not support outer joins")
left_exprs = self.__column_input_to_expression(tuple(left_on) if isinstance(left_on, list) else (left_on,))
right_exprs = self.__column_input_to_expression(tuple(right_on) if isinstance(right_on, list) else (right_on,))
builder = self._builder.join(
other._builder,
left_on=left_exprs,
right_on=right_exprs,
how=join_type,
strategy=join_strategy,
)
return DataFrame(builder)
@DataframePublicAPI
def concat(self, other: "DataFrame") -> "DataFrame":
"""Concatenates two DataFrames together in a "vertical" concatenation. The resulting DataFrame
has number of rows equal to the sum of the number of rows of the input DataFrames.
.. NOTE::
DataFrames being concatenated **must have exactly the same schema**. You may wish to use the
:meth:`df.select() <daft.DataFrame.select>` and :meth:`expr.cast() <daft.Expression.cast>` methods
to ensure schema compatibility before concatenation.
Args:
other (DataFrame): other DataFrame to concatenate
Returns:
DataFrame: DataFrame with rows from `self` on top and rows from `other` at the bottom.
"""