-
Notifications
You must be signed in to change notification settings - Fork 28.1k
/
dataframe.py
2334 lines (2047 loc) · 83.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
#
# 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.
#
# mypy: disable-error-code="override"
from pyspark.errors.exceptions.base import (
SessionNotSameException,
PySparkIndexError,
PySparkAttributeError,
)
from pyspark.resource import ResourceProfile
from pyspark.sql.connect.utils import check_dependencies
check_dependencies(__name__)
from typing import (
Any,
Dict,
Iterator,
List,
Optional,
Tuple,
Union,
Sequence,
TYPE_CHECKING,
overload,
Callable,
cast,
Type,
)
import copy
import sys
import random
import pyarrow as pa
import json
import warnings
from collections.abc import Iterable
import functools
from pyspark import _NoValue
from pyspark._globals import _NoValueType
from pyspark.util import is_remote_only
from pyspark.sql.types import Row, StructType, _create_row
from pyspark.sql.dataframe import (
DataFrame as ParentDataFrame,
DataFrameNaFunctions as ParentDataFrameNaFunctions,
DataFrameStatFunctions as ParentDataFrameStatFunctions,
)
from pyspark.errors import (
PySparkTypeError,
PySparkAttributeError,
PySparkValueError,
PySparkNotImplementedError,
PySparkRuntimeError,
)
from pyspark.util import PythonEvalType
from pyspark.storagelevel import StorageLevel
import pyspark.sql.connect.plan as plan
from pyspark.sql.connect.conversion import ArrowTableToRowsConversion
from pyspark.sql.connect.group import GroupedData
from pyspark.sql.connect.readwriter import DataFrameWriter, DataFrameWriterV2
from pyspark.sql.connect.streaming.readwriter import DataStreamWriter
from pyspark.sql.column import Column
from pyspark.sql.connect.expressions import (
ColumnReference,
UnresolvedRegex,
UnresolvedStar,
)
from pyspark.sql.connect.functions import builtin as F
from pyspark.sql.pandas.types import from_arrow_schema, to_arrow_schema
from pyspark.sql.pandas.functions import _validate_pandas_udf # type: ignore[attr-defined]
if TYPE_CHECKING:
from pyspark.sql.connect._typing import (
ColumnOrName,
ColumnOrNameOrOrdinal,
LiteralType,
PrimitiveType,
OptionalPrimitiveType,
PandasMapIterFunction,
ArrowMapIterFunction,
)
from pyspark.core.rdd import RDD
from pyspark.sql.pandas._typing import DataFrameLike as PandasDataFrameLike
from pyspark.sql.connect.observation import Observation
from pyspark.sql.connect.session import SparkSession
from pyspark.pandas.frame import DataFrame as PandasOnSparkDataFrame
from pyspark.sql.metrics import ExecutionInfo
class DataFrame(ParentDataFrame):
def __new__(
cls,
plan: plan.LogicalPlan,
session: "SparkSession",
) -> "DataFrame":
self = object.__new__(cls)
self.__init__(plan, session) # type: ignore[misc]
return self
def __init__(
self,
plan: plan.LogicalPlan,
session: "SparkSession",
):
"""Creates a new data frame"""
self._plan = plan
if self._plan is None:
raise PySparkRuntimeError(
error_class="MISSING_VALID_PLAN",
message_parameters={"operator": "__init__"},
)
self._session: "SparkSession" = session # type: ignore[assignment]
if self._session is None:
raise PySparkRuntimeError(
error_class="NO_ACTIVE_SESSION",
message_parameters={"operator": "__init__"},
)
# Check whether _repr_html is supported or not, we use it to avoid calling RPC twice
# by __repr__ and _repr_html_ while eager evaluation opens.
self._support_repr_html = False
self._cached_schema: Optional[StructType] = None
self._execution_info: Optional["ExecutionInfo"] = None
def __reduce__(self) -> Tuple:
"""
Custom method for serializing the DataFrame object using Pickle. Since the DataFrame
overrides "__getattr__" method, the default serialization method does not work.
Returns
-------
The tuple containing the information needed to reconstruct the object.
"""
return (
DataFrame,
(
self._plan,
self._session,
),
{
"_support_repr_html": self._support_repr_html,
"_cached_schema": self._cached_schema,
},
)
def __repr__(self) -> str:
if not self._support_repr_html:
(
repl_eager_eval_enabled,
repl_eager_eval_max_num_rows,
repl_eager_eval_truncate,
) = self._session._client.get_configs(
"spark.sql.repl.eagerEval.enabled",
"spark.sql.repl.eagerEval.maxNumRows",
"spark.sql.repl.eagerEval.truncate",
)
if repl_eager_eval_enabled == "true":
return self._show_string(
n=int(cast(str, repl_eager_eval_max_num_rows)),
truncate=int(cast(str, repl_eager_eval_truncate)),
vertical=False,
)
return "DataFrame[%s]" % (", ".join("%s: %s" % c for c in self.dtypes))
def _repr_html_(self) -> Optional[str]:
if not self._support_repr_html:
self._support_repr_html = True
(
repl_eager_eval_enabled,
repl_eager_eval_max_num_rows,
repl_eager_eval_truncate,
) = self._session._client.get_configs(
"spark.sql.repl.eagerEval.enabled",
"spark.sql.repl.eagerEval.maxNumRows",
"spark.sql.repl.eagerEval.truncate",
)
if repl_eager_eval_enabled == "true":
table, _ = DataFrame(
plan.HtmlString(
child=self._plan,
num_rows=int(cast(str, repl_eager_eval_max_num_rows)),
truncate=int(cast(str, repl_eager_eval_truncate)),
),
session=self._session,
)._to_table()
return table[0][0].as_py()
else:
return None
@property
def write(self) -> "DataFrameWriter":
def cb(qe: "ExecutionInfo") -> None:
self._execution_info = qe
return DataFrameWriter(self._plan, self._session, cb)
@functools.cache
def isEmpty(self) -> bool:
return len(self.select().take(1)) == 0
@overload
def select(self, *cols: "ColumnOrName") -> ParentDataFrame:
...
@overload
def select(self, __cols: Union[List[Column], List[str]]) -> ParentDataFrame:
...
def select(self, *cols: "ColumnOrName") -> ParentDataFrame: # type: ignore[misc]
if len(cols) == 1 and isinstance(cols[0], list):
cols = cols[0]
if any(not isinstance(c, (str, Column)) for c in cols):
raise PySparkTypeError(
error_class="NOT_LIST_OF_COLUMN_OR_STR",
message_parameters={"arg_name": "columns"},
)
return DataFrame(
plan.Project(self._plan, [F._to_col(c) for c in cols]),
session=self._session,
)
def selectExpr(self, *expr: Union[str, List[str]]) -> ParentDataFrame:
sql_expr = []
if len(expr) == 1 and isinstance(expr[0], list):
expr = expr[0] # type: ignore[assignment]
for element in expr:
if isinstance(element, str):
sql_expr.append(F.expr(element))
else:
sql_expr.extend([F.expr(e) for e in element])
return DataFrame(plan.Project(self._plan, sql_expr), session=self._session)
def agg(self, *exprs: Union[Column, Dict[str, str]]) -> ParentDataFrame:
if not exprs:
raise PySparkValueError(
error_class="CANNOT_BE_EMPTY",
message_parameters={"item": "exprs"},
)
if len(exprs) == 1 and isinstance(exprs[0], dict):
measures = [F._invoke_function(f, F.col(e)) for e, f in exprs[0].items()]
return self.groupBy().agg(*measures)
else:
# other expressions
assert all(isinstance(c, Column) for c in exprs), "all exprs should be Expression"
exprs = cast(Tuple[Column, ...], exprs)
return self.groupBy().agg(*exprs)
def alias(self, alias: str) -> ParentDataFrame:
res = DataFrame(plan.SubqueryAlias(self._plan, alias), session=self._session)
res._cached_schema = self._cached_schema
return res
def colRegex(self, colName: str) -> Column:
from pyspark.sql.connect.column import Column as ConnectColumn
if not isinstance(colName, str):
raise PySparkTypeError(
error_class="NOT_STR",
message_parameters={"arg_name": "colName", "arg_type": type(colName).__name__},
)
return ConnectColumn(UnresolvedRegex(colName, self._plan._plan_id))
@property
def dtypes(self) -> List[Tuple[str, str]]:
return [(str(f.name), f.dataType.simpleString()) for f in self.schema.fields]
@property
def columns(self) -> List[str]:
return self.schema.names
@property
def sparkSession(self) -> "SparkSession":
return self._session
def count(self) -> int:
table, _ = self.agg(
F._invoke_function("count", F.lit(1))
)._to_table() # type: ignore[operator]
return table[0][0].as_py()
def crossJoin(self, other: ParentDataFrame) -> ParentDataFrame:
self._check_same_session(other)
return DataFrame(
plan.Join(
left=self._plan, right=other._plan, on=None, how="cross" # type: ignore[arg-type]
),
session=self._session,
)
def _check_same_session(self, other: ParentDataFrame) -> None:
if self._session.session_id != other._session.session_id: # type: ignore[attr-defined]
raise SessionNotSameException(
error_class="SESSION_NOT_SAME",
message_parameters={},
)
def coalesce(self, numPartitions: int) -> ParentDataFrame:
if not numPartitions > 0:
raise PySparkValueError(
error_class="VALUE_NOT_POSITIVE",
message_parameters={"arg_name": "numPartitions", "arg_value": str(numPartitions)},
)
res = DataFrame(
plan.Repartition(self._plan, num_partitions=numPartitions, shuffle=False),
self._session,
)
res._cached_schema = self._cached_schema
return res
@overload
def repartition(self, numPartitions: int, *cols: "ColumnOrName") -> ParentDataFrame:
...
@overload
def repartition(self, *cols: "ColumnOrName") -> ParentDataFrame:
...
def repartition( # type: ignore[misc]
self, numPartitions: Union[int, "ColumnOrName"], *cols: "ColumnOrName"
) -> ParentDataFrame:
if isinstance(numPartitions, int):
if not numPartitions > 0:
raise PySparkValueError(
error_class="VALUE_NOT_POSITIVE",
message_parameters={
"arg_name": "numPartitions",
"arg_value": str(numPartitions),
},
)
if len(cols) == 0:
res = DataFrame(
plan.Repartition(self._plan, numPartitions, shuffle=True),
self._session,
)
else:
res = DataFrame(
plan.RepartitionByExpression(
self._plan, numPartitions, [F._to_col(c) for c in cols]
),
self.sparkSession,
)
elif isinstance(numPartitions, (str, Column)):
cols = (numPartitions,) + cols
res = DataFrame(
plan.RepartitionByExpression(self._plan, None, [F._to_col(c) for c in cols]),
self.sparkSession,
)
else:
raise PySparkTypeError(
error_class="NOT_COLUMN_OR_STR",
message_parameters={
"arg_name": "numPartitions",
"arg_type": type(numPartitions).__name__,
},
)
res._cached_schema = self._cached_schema
return res
@overload
def repartitionByRange(self, numPartitions: int, *cols: "ColumnOrName") -> ParentDataFrame:
...
@overload
def repartitionByRange(self, *cols: "ColumnOrName") -> ParentDataFrame:
...
def repartitionByRange( # type: ignore[misc]
self, numPartitions: Union[int, "ColumnOrName"], *cols: "ColumnOrName"
) -> ParentDataFrame:
if isinstance(numPartitions, int):
if not numPartitions > 0:
raise PySparkValueError(
error_class="VALUE_NOT_POSITIVE",
message_parameters={
"arg_name": "numPartitions",
"arg_value": str(numPartitions),
},
)
if len(cols) == 0:
raise PySparkValueError(
error_class="CANNOT_BE_EMPTY",
message_parameters={"item": "cols"},
)
else:
res = DataFrame(
plan.RepartitionByExpression(
self._plan, numPartitions, [F._sort_col(c) for c in cols]
),
self.sparkSession,
)
elif isinstance(numPartitions, (str, Column)):
res = DataFrame(
plan.RepartitionByExpression(
self._plan, None, [F._sort_col(c) for c in [numPartitions] + list(cols)]
),
self.sparkSession,
)
else:
raise PySparkTypeError(
error_class="NOT_COLUMN_OR_INT_OR_STR",
message_parameters={
"arg_name": "numPartitions",
"arg_type": type(numPartitions).__name__,
},
)
res._cached_schema = self._cached_schema
return res
def dropDuplicates(self, *subset: Union[str, List[str]]) -> ParentDataFrame:
# Acceptable args should be str, ... or a single List[str]
# So if subset length is 1, it can be either single str, or a list of str
# if subset length is greater than 1, it must be a sequence of str
if len(subset) > 1:
assert all(isinstance(c, str) for c in subset)
if not subset:
res = DataFrame(
plan.Deduplicate(child=self._plan, all_columns_as_keys=True), session=self._session
)
elif len(subset) == 1 and isinstance(subset[0], list):
res = DataFrame(
plan.Deduplicate(child=self._plan, column_names=subset[0]),
session=self._session,
)
else:
res = DataFrame(
plan.Deduplicate(child=self._plan, column_names=cast(List[str], subset)),
session=self._session,
)
res._cached_schema = self._cached_schema
return res
drop_duplicates = dropDuplicates
def dropDuplicatesWithinWatermark(self, *subset: Union[str, List[str]]) -> ParentDataFrame:
# Acceptable args should be str, ... or a single List[str]
# So if subset length is 1, it can be either single str, or a list of str
# if subset length is greater than 1, it must be a sequence of str
if len(subset) > 1:
assert all(isinstance(c, str) for c in subset)
if not subset:
return DataFrame(
plan.Deduplicate(child=self._plan, all_columns_as_keys=True, within_watermark=True),
session=self._session,
)
elif len(subset) == 1 and isinstance(subset[0], list):
return DataFrame(
plan.Deduplicate(child=self._plan, column_names=subset[0], within_watermark=True),
session=self._session,
)
else:
return DataFrame(
plan.Deduplicate(
child=self._plan,
column_names=cast(List[str], subset),
within_watermark=True,
),
session=self._session,
)
def distinct(self) -> ParentDataFrame:
res = DataFrame(
plan.Deduplicate(child=self._plan, all_columns_as_keys=True), session=self._session
)
res._cached_schema = self._cached_schema
return res
@overload
def drop(self, cols: "ColumnOrName") -> ParentDataFrame:
...
@overload
def drop(self, *cols: str) -> ParentDataFrame:
...
def drop(self, *cols: "ColumnOrName") -> ParentDataFrame: # type: ignore[misc]
_cols = list(cols)
if any(not isinstance(c, (str, Column)) for c in _cols):
raise PySparkTypeError(
error_class="NOT_COLUMN_OR_STR",
message_parameters={"arg_name": "cols", "arg_type": type(cols).__name__},
)
return DataFrame(
plan.Drop(
child=self._plan,
columns=_cols,
),
session=self._session,
)
def filter(self, condition: Union[Column, str]) -> ParentDataFrame:
if isinstance(condition, str):
expr = F.expr(condition)
else:
expr = condition
res = DataFrame(plan.Filter(child=self._plan, filter=expr), session=self._session)
res._cached_schema = self._cached_schema
return res
def first(self) -> Optional[Row]:
return self.head()
@overload # type: ignore[no-overload-impl]
def groupby(self, *cols: "ColumnOrNameOrOrdinal") -> "GroupedData":
...
@overload
def groupby(self, __cols: Union[List[Column], List[str], List[int]]) -> "GroupedData":
...
def groupBy(self, *cols: "ColumnOrNameOrOrdinal") -> GroupedData:
if len(cols) == 1 and isinstance(cols[0], list):
cols = cols[0]
_cols: List[Column] = []
for c in cols:
if isinstance(c, Column):
_cols.append(c)
elif isinstance(c, str):
_cols.append(self[c])
elif isinstance(c, int) and not isinstance(c, bool):
if c < 1:
raise PySparkIndexError(
error_class="INDEX_NOT_POSITIVE", message_parameters={"index": str(c)}
)
# ordinal is 1-based
_cols.append(self[c - 1])
else:
raise PySparkTypeError(
error_class="NOT_COLUMN_OR_STR",
message_parameters={"arg_name": "cols", "arg_type": type(c).__name__},
)
return GroupedData(df=self, group_type="groupby", grouping_cols=_cols)
groupby = groupBy # type: ignore[assignment]
@overload
def rollup(self, *cols: "ColumnOrName") -> "GroupedData":
...
@overload
def rollup(self, __cols: Union[List[Column], List[str]]) -> "GroupedData":
...
def rollup(self, *cols: "ColumnOrName") -> "GroupedData": # type: ignore[misc]
_cols: List[Column] = []
for c in cols:
if isinstance(c, Column):
_cols.append(c)
elif isinstance(c, str):
_cols.append(self[c])
elif isinstance(c, int) and not isinstance(c, bool):
if c < 1:
raise PySparkIndexError(
error_class="INDEX_NOT_POSITIVE", message_parameters={"index": str(c)}
)
# ordinal is 1-based
_cols.append(self[c - 1])
else:
raise PySparkTypeError(
error_class="NOT_COLUMN_OR_STR",
message_parameters={"arg_name": "cols", "arg_type": type(c).__name__},
)
return GroupedData(df=self, group_type="rollup", grouping_cols=_cols)
@overload
def cube(self, *cols: "ColumnOrName") -> "GroupedData":
...
@overload
def cube(self, __cols: Union[List[Column], List[str]]) -> "GroupedData":
...
def cube(self, *cols: "ColumnOrName") -> "GroupedData": # type: ignore[misc]
_cols: List[Column] = []
for c in cols:
if isinstance(c, Column):
_cols.append(c)
elif isinstance(c, str):
_cols.append(self[c])
elif isinstance(c, int) and not isinstance(c, bool):
if c < 1:
raise PySparkIndexError(
error_class="INDEX_NOT_POSITIVE", message_parameters={"index": str(c)}
)
# ordinal is 1-based
_cols.append(self[c - 1])
else:
raise PySparkTypeError(
error_class="NOT_COLUMN_OR_STR",
message_parameters={"arg_name": "cols", "arg_type": type(c).__name__},
)
return GroupedData(df=self, group_type="cube", grouping_cols=_cols)
def groupingSets(
self, groupingSets: Sequence[Sequence["ColumnOrName"]], *cols: "ColumnOrName"
) -> "GroupedData":
gsets: List[List[Column]] = []
for grouping_set in groupingSets:
gset: List[Column] = []
for c in grouping_set:
if isinstance(c, Column):
gset.append(c)
elif isinstance(c, str):
gset.append(self[c])
else:
raise PySparkTypeError(
error_class="NOT_COLUMN_OR_STR",
message_parameters={
"arg_name": "groupingSets",
"arg_type": type(c).__name__,
},
)
gsets.append(gset)
gcols: List[Column] = []
for c in cols:
if isinstance(c, Column):
gcols.append(c)
elif isinstance(c, str):
gcols.append(self[c])
else:
raise PySparkTypeError(
error_class="NOT_COLUMN_OR_STR",
message_parameters={"arg_name": "cols", "arg_type": type(c).__name__},
)
return GroupedData(
df=self, group_type="grouping_sets", grouping_cols=gcols, grouping_sets=gsets
)
@overload
def head(self) -> Optional[Row]:
...
@overload
def head(self, n: int) -> List[Row]:
...
def head(self, n: Optional[int] = None) -> Union[Optional[Row], List[Row]]:
if n is None:
rs = self.head(1)
return rs[0] if rs else None
return self.take(n)
def take(self, num: int) -> List[Row]:
return self.limit(num).collect()
def join(
self,
other: ParentDataFrame,
on: Optional[Union[str, List[str], Column, List[Column]]] = None,
how: Optional[str] = None,
) -> ParentDataFrame:
self._check_same_session(other)
if how is not None and isinstance(how, str):
how = how.lower().replace("_", "")
return DataFrame(
plan.Join(left=self._plan, right=other._plan, on=on, how=how), # type: ignore[arg-type]
session=self._session,
)
def _joinAsOf(
self,
other: ParentDataFrame,
leftAsOfColumn: Union[str, Column],
rightAsOfColumn: Union[str, Column],
on: Optional[Union[str, List[str], Column, List[Column]]] = None,
how: Optional[str] = None,
*,
tolerance: Optional[Column] = None,
allowExactMatches: bool = True,
direction: str = "backward",
) -> ParentDataFrame:
self._check_same_session(other)
if how is None:
how = "inner"
assert isinstance(how, str), "how should be a string"
if tolerance is not None:
assert isinstance(tolerance, Column), "tolerance should be Column"
def _convert_col(df: ParentDataFrame, col: "ColumnOrName") -> Column:
if isinstance(col, Column):
return col
else:
return df._col(col) # type: ignore[operator]
return DataFrame(
plan.AsOfJoin(
left=self._plan,
right=other._plan, # type: ignore[arg-type]
left_as_of=_convert_col(self, leftAsOfColumn),
right_as_of=_convert_col(other, rightAsOfColumn),
on=on,
how=how,
tolerance=tolerance,
allow_exact_matches=allowExactMatches,
direction=direction,
),
session=self._session,
)
def limit(self, n: int) -> ParentDataFrame:
res = DataFrame(plan.Limit(child=self._plan, limit=n), session=self._session)
res._cached_schema = self._cached_schema
return res
def tail(self, num: int) -> List[Row]:
return DataFrame(plan.Tail(child=self._plan, limit=num), session=self._session).collect()
def _sort_cols(
self,
cols: Sequence[Union[int, str, Column, List[Union[int, str, Column]]]],
kwargs: Dict[str, Any],
) -> List[Column]:
"""Return a JVM Seq of Columns that describes the sort order"""
if cols is None:
raise PySparkValueError(
error_class="CANNOT_BE_EMPTY",
message_parameters={"item": "cols"},
)
if len(cols) == 1 and isinstance(cols[0], list):
cols = cols[0]
_cols: List[Column] = []
for c in cols:
if isinstance(c, int) and not isinstance(c, bool):
# ordinal is 1-based
if c > 0:
_c = self[c - 1]
# negative ordinal means sort by desc
elif c < 0:
_c = self[-c - 1].desc()
else:
raise PySparkIndexError(
error_class="ZERO_INDEX",
message_parameters={},
)
else:
_c = c # type: ignore[assignment]
_cols.append(F._to_col(cast("ColumnOrName", _c)))
ascending = kwargs.get("ascending", True)
if isinstance(ascending, (bool, int)):
if not ascending:
_cols = [c.desc() for c in _cols]
elif isinstance(ascending, list):
_cols = [c if asc else c.desc() for asc, c in zip(ascending, _cols)]
else:
raise PySparkTypeError(
error_class="NOT_BOOL_OR_LIST",
message_parameters={"arg_name": "ascending", "arg_type": type(ascending).__name__},
)
return [F._sort_col(c) for c in _cols]
def sort(
self,
*cols: Union[int, str, Column, List[Union[int, str, Column]]],
**kwargs: Any,
) -> ParentDataFrame:
res = DataFrame(
plan.Sort(
self._plan,
columns=self._sort_cols(cols, kwargs),
is_global=True,
),
session=self._session,
)
res._cached_schema = self._cached_schema
return res
orderBy = sort
def sortWithinPartitions(
self,
*cols: Union[int, str, Column, List[Union[int, str, Column]]],
**kwargs: Any,
) -> ParentDataFrame:
res = DataFrame(
plan.Sort(
self._plan,
columns=self._sort_cols(cols, kwargs),
is_global=False,
),
session=self._session,
)
res._cached_schema = self._cached_schema
return res
def sample(
self,
withReplacement: Optional[Union[float, bool]] = None,
fraction: Optional[Union[int, float]] = None,
seed: Optional[int] = None,
) -> ParentDataFrame:
# For the cases below:
# sample(True, 0.5 [, seed])
# sample(True, fraction=0.5 [, seed])
# sample(withReplacement=False, fraction=0.5 [, seed])
is_withReplacement_set = type(withReplacement) == bool and isinstance(fraction, float)
# For the case below:
# sample(faction=0.5 [, seed])
is_withReplacement_omitted_kwargs = withReplacement is None and isinstance(fraction, float)
# For the case below:
# sample(0.5 [, seed])
is_withReplacement_omitted_args = isinstance(withReplacement, float)
if not (
is_withReplacement_set
or is_withReplacement_omitted_kwargs
or is_withReplacement_omitted_args
):
argtypes = [type(arg).__name__ for arg in [withReplacement, fraction, seed]]
raise PySparkTypeError(
error_class="NOT_BOOL_OR_FLOAT_OR_INT",
message_parameters={
"arg_name": "withReplacement (optional), "
+ "fraction (required) and seed (optional)",
"arg_type": ", ".join(argtypes),
},
)
if is_withReplacement_omitted_args:
if fraction is not None:
seed = cast(int, fraction)
fraction = withReplacement
withReplacement = None
if withReplacement is None:
withReplacement = False
seed = int(seed) if seed is not None else random.randint(0, sys.maxsize)
res = DataFrame(
plan.Sample(
child=self._plan,
lower_bound=0.0,
upper_bound=fraction, # type: ignore[arg-type]
with_replacement=withReplacement, # type: ignore[arg-type]
seed=seed,
),
session=self._session,
)
res._cached_schema = self._cached_schema
return res
def withColumnRenamed(self, existing: str, new: str) -> ParentDataFrame:
return self.withColumnsRenamed({existing: new})
def withColumnsRenamed(self, colsMap: Dict[str, str]) -> ParentDataFrame:
if not isinstance(colsMap, dict):
raise PySparkTypeError(
error_class="NOT_DICT",
message_parameters={"arg_name": "colsMap", "arg_type": type(colsMap).__name__},
)
return DataFrame(plan.WithColumnsRenamed(self._plan, colsMap), self._session)
def _show_string(
self, n: int = 20, truncate: Union[bool, int] = True, vertical: bool = False
) -> str:
if not isinstance(n, int) or isinstance(n, bool):
raise PySparkTypeError(
error_class="NOT_INT",
message_parameters={"arg_name": "n", "arg_type": type(n).__name__},
)
if not isinstance(vertical, bool):
raise PySparkTypeError(
error_class="NOT_BOOL",
message_parameters={"arg_name": "vertical", "arg_type": type(vertical).__name__},
)
_truncate: int = -1
if isinstance(truncate, bool) and truncate:
_truncate = 20
else:
try:
_truncate = int(truncate)
except ValueError:
raise PySparkTypeError(
error_class="NOT_BOOL",
message_parameters={
"arg_name": "truncate",
"arg_type": type(truncate).__name__,
},
)
table, _ = DataFrame(
plan.ShowString(
child=self._plan,
num_rows=n,
truncate=_truncate,
vertical=vertical,
),
session=self._session,
)._to_table()
return table[0][0].as_py()
def withColumns(self, colsMap: Dict[str, Column]) -> ParentDataFrame:
if not isinstance(colsMap, dict):
raise PySparkTypeError(
error_class="NOT_DICT",
message_parameters={"arg_name": "colsMap", "arg_type": type(colsMap).__name__},
)
names: List[str] = []
columns: List[Column] = []
for columnName, column in colsMap.items():
names.append(columnName)
columns.append(column)
return DataFrame(
plan.WithColumns(
self._plan,
columnNames=names,
columns=columns,
),
session=self._session,
)
def withColumn(self, colName: str, col: Column) -> ParentDataFrame:
if not isinstance(col, Column):
raise PySparkTypeError(
error_class="NOT_COLUMN",
message_parameters={"arg_name": "col", "arg_type": type(col).__name__},
)
return DataFrame(
plan.WithColumns(
self._plan,
columnNames=[colName],
columns=[col],
),
session=self._session,
)
def withMetadata(self, columnName: str, metadata: Dict[str, Any]) -> ParentDataFrame:
if not isinstance(metadata, dict):
raise PySparkTypeError(
error_class="NOT_DICT",
message_parameters={"arg_name": "metadata", "arg_type": type(metadata).__name__},
)
return DataFrame(
plan.WithColumns(
self._plan,
columnNames=[columnName],
columns=[self[columnName]],
metadata=[json.dumps(metadata)],
),
session=self._session,
)
def unpivot(
self,
ids: Union["ColumnOrName", List["ColumnOrName"], Tuple["ColumnOrName", ...]],
values: Optional[Union["ColumnOrName", List["ColumnOrName"], Tuple["ColumnOrName", ...]]],
variableColumnName: str,
valueColumnName: str,
) -> ParentDataFrame:
assert ids is not None, "ids must not be None"
def _convert_cols(
cols: Optional[Union["ColumnOrName", List["ColumnOrName"], Tuple["ColumnOrName", ...]]]
) -> List[Column]:
if cols is None:
return []