Skip to content
25 changes: 14 additions & 11 deletions native/spark-expr/src/math_funcs/internal/make_decimal.rs
Original file line number Diff line number Diff line change
Expand Up @@ -40,18 +40,21 @@ pub fn spark_make_decimal(
))),
sv => internal_err!("Expected Int64 but found {sv:?}"),
},
ColumnarValue::Array(a) => {
let arr = a.as_primitive::<Int64Type>();
let mut result = Decimal128Builder::new();
for v in arr.into_iter() {
result.append_option(long_to_decimal(&v, precision))
}
let result_type = DataType::Decimal128(precision, scale);
ColumnarValue::Array(a) => match a.data_type() {
DataType::Int64 => {
let arr = a.as_primitive::<Int64Type>();
let mut result = Decimal128Builder::new();
for v in arr.into_iter() {
result.append_option(long_to_decimal(&v, precision))
}
let result_type = DataType::Decimal128(precision, scale);

Ok(ColumnarValue::Array(Arc::new(
result.finish().with_data_type(result_type),
)))
}
Ok(ColumnarValue::Array(Arc::new(
result.finish().with_data_type(result_type),
)))
}
av => internal_err!("Expected Int64 but found {av:?}"),
},
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,14 @@ object CometUnscaledValue extends CometExpressionSerde[UnscaledValue] {
}

object CometMakeDecimal extends CometExpressionSerde[MakeDecimal] {

override def getSupportLevel(expr: MakeDecimal): SupportLevel = {
expr.child.dataType match {
case LongType => Compatible()
case other => Unsupported(Some(s"Unsupported input data type: $other"))
}
}

override def convert(
expr: MakeDecimal,
inputs: Seq[Attribute],
Expand Down
40 changes: 40 additions & 0 deletions spark/src/test/scala/org/apache/comet/CometExpressionSuite.scala
Original file line number Diff line number Diff line change
Expand Up @@ -3111,4 +3111,44 @@ class CometExpressionSuite extends CometTestBase with AdaptiveSparkPlanHelper {
CometConcat.unsupportedReason)
}
}

// https://github.com/apache/datafusion-comet/issues/2813
test("make decimal using DataFrame API - integer") {
withTable("t1") {
sql("create table t1 using parquet as select 123456 as c1 from range(1)")

withSQLConf(
SQLConf.USE_V1_SOURCE_LIST.key -> "parquet",
SQLConf.ANSI_ENABLED.key -> "false",
SQLConf.ADAPTIVE_EXECUTION_ENABLED.key -> "false",
SQLConf.ADAPTIVE_OPTIMIZER_EXCLUDED_RULES.key -> "org.apache.spark.sql.catalyst.optimizer.ConstantFolding") {
Comment on lines +3120 to +3124
Copy link
Copy Markdown

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

⚠️ Potential issue | 🟡 Minor

🧩 Analysis chain

🏁 Script executed:

#!/bin/bash
# Read the test code around lines 3120-3124
sed -n '3100,3150p' spark/src/test/scala/org/apache/comet/CometExpressionSuite.scala

Repository: martin-augment/datafusion-comet

Length of output: 2133


🏁 Script executed:

#!/bin/bash
# Search for the test function name to understand the context better
sed -n '3000,3150p' spark/src/test/scala/org/apache/comet/CometExpressionSuite.scala | head -100

Repository: martin-augment/datafusion-comet

Length of output: 4224


🏁 Script executed:

#!/bin/bash
# Search for MakeDecimal in the test file to see related patterns
rg -n "MakeDecimal" spark/src/test/scala/org/apache/comet/CometExpressionSuite.scala -B 5 -A 15

Repository: martin-augment/datafusion-comet

Length of output: 1466


🏁 Script executed:

#!/bin/bash
# Search for similar test patterns using withSQLConf and these specific configs
rg -n "ADAPTIVE_EXECUTION_ENABLED.*false" spark/src/test/scala/ -B 3 -A 3

Repository: martin-augment/datafusion-comet

Length of output: 8410


🏁 Script executed:

#!/bin/bash
# Check if there are comments or documentation about why these configs are needed
rg -n "ConstantFolding\|ADAPTIVE_EXECUTION_ENABLED" spark/src/test/scala/org/apache/comet/ -B 2 -A 2

Repository: martin-augment/datafusion-comet

Length of output: 57


🏁 Script executed:

#!/bin/bash
# Search for other MakeDecimal tests and their configs
rg -n "MakeDecimal\|makeDecimal" spark/src/test/scala/ -B 10 -A 5 | grep -A 10 -B 10 "withSQLConf"

Repository: martin-augment/datafusion-comet

Length of output: 57


🏁 Script executed:

#!/bin/bash
# Look for any comments or documentation about ConstantFolding
rg -n "ConstantFolding" spark/src/test/scala/org/apache/comet/ -B 3 -A 3

Repository: martin-augment/datafusion-comet

Length of output: 8481


🏁 Script executed:

#!/bin/bash
# Check the createMakeDecimalColumn function to understand what it does
rg -n "def createMakeDecimalColumn" spark/src/test/scala/ -A 10

Repository: martin-augment/datafusion-comet

Length of output: 57


🏁 Script executed:

#!/bin/bash
# Look for the issue 2813 or any comments linking to it
rg -n "2813" spark/src/test/scala/org/apache/comet/

Repository: martin-augment/datafusion-comet

Length of output: 204


Add comments explaining the SQL configuration settings for MakeDecimal tests.

These configurations are necessary for testing MakeDecimal with scalar values but lack documentation. Other similar tests in the file (e.g., lines 1437–1438) include explanatory comments:

  • ADAPTIVE_OPTIMIZER_EXCLUDED_RULES with ConstantFolding: Prevents Spark from pre-computing literal expressions, allowing actual MakeDecimal execution to be tested
  • ADAPTIVE_EXECUTION_ENABLED -> "false": Disables adaptive query execution to avoid plan restructuring that could interfere with testing

Add similar explanatory comments to lines 3120–3124 and 3139–3143 (the "long" variant test).

🤖 Prompt for AI Agents
In spark/src/test/scala/org/apache/comet/CometExpressionSuite.scala around lines
3120–3124 and also update the analogous block at lines 3139–3143, add inline
comments explaining each SQLConf used for the MakeDecimal scalar tests: note
that ADAPTIVE_OPTIMIZER_EXCLUDED_RULES set to ConstantFolding prevents Spark
from pre-computing literal expressions so MakeDecimal is executed,
ADAPTIVE_EXECUTION_ENABLED -> "false" disables adaptive query execution to avoid
plan restructuring, ANSI_ENABLED -> "false" and USE_V1_SOURCE_LIST -> "parquet"
are set for deterministic behavior with parquet literals; place these brief
explanatory comments immediately above or beside the withSQLConf block in both
locations.

Copy link
Copy Markdown
Owner Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

value:good-to-have; category:bug; feedback:The CodeRabbit AI reviewer is correct! For easier maintenance in the future it will be good to add a short comment next to each setting to explain why it is needed for this particular test. Prevents wasted time debugging such test in the future.


val df = sql("select * from t1")
val makeDecimalColumn = createMakeDecimalColumn(df.col("c1").expr, 3, 0)
val df1 = df.withColumn("result", makeDecimalColumn)

checkSparkAnswerAndFallbackReason(df1, "Unsupported input data type: IntegerType")
}
}
}

test("make decimal using DataFrame API - long") {
withTable("t1") {
sql("create table t1 using parquet as select cast(123456 as long) as c1 from range(1)")

withSQLConf(
SQLConf.USE_V1_SOURCE_LIST.key -> "parquet",
SQLConf.ANSI_ENABLED.key -> "false",
SQLConf.ADAPTIVE_EXECUTION_ENABLED.key -> "false",
SQLConf.ADAPTIVE_OPTIMIZER_EXCLUDED_RULES.key -> "org.apache.spark.sql.catalyst.optimizer.ConstantFolding") {

val df = sql("select * from t1")
val makeDecimalColumn = createMakeDecimalColumn(df.col("c1").expr, 3, 0)
val df1 = df.withColumn("result", makeDecimalColumn)

checkSparkAnswerAndOperator(df1)
}
}
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -20,7 +20,7 @@
package org.apache.spark.sql

import org.apache.spark.SparkConf
import org.apache.spark.sql.catalyst.expressions.Expression
import org.apache.spark.sql.catalyst.expressions.{Expression, MakeDecimal}
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan

trait ShimCometTestBase {
Expand All @@ -46,4 +46,8 @@ trait ShimCometTestBase {
def extractLogicalPlan(df: DataFrame): LogicalPlan = {
df.logicalPlan
}

def createMakeDecimalColumn(child: Expression, precision: Int, scale: Int): Column = {
new Column(MakeDecimal(child, precision, scale))
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -20,7 +20,7 @@
package org.apache.spark.sql

import org.apache.spark.SparkConf
import org.apache.spark.sql.catalyst.expressions.Expression
import org.apache.spark.sql.catalyst.expressions.{Expression, MakeDecimal}
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan

trait ShimCometTestBase {
Expand All @@ -47,4 +47,8 @@ trait ShimCometTestBase {
df.logicalPlan
}

def createMakeDecimalColumn(child: Expression, precision: Int, scale: Int): Column = {
new Column(MakeDecimal(child, precision, scale))
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -20,7 +20,7 @@
package org.apache.spark.sql

import org.apache.spark.SparkConf
import org.apache.spark.sql.catalyst.expressions.Expression
import org.apache.spark.sql.catalyst.expressions.{Expression, MakeDecimal}
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
import org.apache.spark.sql.classic.{Dataset, ExpressionColumnNode, SparkSession}

Expand All @@ -47,4 +47,8 @@ trait ShimCometTestBase {
def extractLogicalPlan(df: DataFrame): LogicalPlan = {
df.queryExecution.analyzed
}

def createMakeDecimalColumn(child: Expression, precision: Int, scale: Int): Column = {
new Column(ExpressionColumnNode.apply(MakeDecimal(child, precision, scale, true)))
}
}