Skip to content

[SPARK-52528][PS] Enable divide-by-zero for numeric mod with ANSI enabled #51219

New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Open
wants to merge 2 commits into
base: master
Choose a base branch
from
Open
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
8 changes: 7 additions & 1 deletion python/pyspark/pandas/data_type_ops/num_ops.py
Original file line number Diff line number Diff line change
Expand Up @@ -96,9 +96,15 @@ def mod(self, left: IndexOpsLike, right: Any) -> SeriesOrIndex:
_sanitize_list_like(right)
if not is_valid_operand_for_numeric_arithmetic(right):
raise TypeError("Modulo can not be applied to given types.")
spark_session = left._internal.spark_frame.sparkSession

def mod(left: PySparkColumn, right: Any) -> PySparkColumn:
return ((left % right) + right) % right
if is_ansi_mode_enabled(spark_session):
return F.when(F.lit(right == 0), F.lit(None)).otherwise(
((left % right) + right) % right
)
else:
return ((left % right) + right) % right

right = transform_boolean_operand_to_numeric(right, spark_type=left.spark.data_type)
return column_op(mod)(left, right)
Expand Down
1 change: 1 addition & 0 deletions python/pyspark/pandas/tests/computation/test_binary_ops.py
Original file line number Diff line number Diff line change
Expand Up @@ -225,6 +225,7 @@ def test_binary_operator_mod(self):
psdf = ps.from_pandas(pdf)

self.assert_eq(psdf["a"] % psdf["b"], pdf["a"] % pdf["b"])
self.assert_eq(psdf["a"] % 0, pdf["a"] % 0)

# Negative
psdf = ps.DataFrame({"a": ["x"], "b": [1]})
Expand Down
1 change: 0 additions & 1 deletion python/pyspark/pandas/tests/series/test_stat.py
Original file line number Diff line number Diff line change
Expand Up @@ -443,7 +443,6 @@ def test_rdivmod(self):
self.assert_eq(krdiv, prdiv)
self.assert_eq(krmod, prmod)

@unittest.skipIf(is_ansi_mode_test, ansi_mode_not_supported_message)
def test_mod(self):
pser = pd.Series([100, None, -300, None, 500, -700], name="Koalas")
psser = ps.from_pandas(pser)
Expand Down
1 change: 0 additions & 1 deletion python/pyspark/pandas/tests/test_numpy_compat.py
Original file line number Diff line number Diff line change
Expand Up @@ -132,7 +132,6 @@ def test_np_spark_compat_series(self):
finally:
reset_option("compute.ops_on_diff_frames")

@unittest.skipIf(is_ansi_mode_test, ansi_mode_not_supported_message)
def test_np_spark_compat_frame(self):
from pyspark.pandas.numpy_compat import unary_np_spark_mappings, binary_np_spark_mappings

Expand Down