Skip to content

Commit

Permalink
Spark 3.4 UDF: change pmod to mod because positiveModulo does not exi…
Browse files Browse the repository at this point in the history
…sts in early version of clickhouse

(cherry picked from commit ea0592d)
(cherry picked from commit 8a270a2)
  • Loading branch information
Yxang committed Jul 24, 2023
1 parent fb3bf50 commit d9313e6
Show file tree
Hide file tree
Showing 4 changed files with 7 additions and 10 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -108,7 +108,7 @@ class ClusterShardByTransformSuite extends SparkClickHouseClusterTest {
("murmurHash3_64", Array("value")),
("murmurHash3_32", Array("value")),
("cityHash64", Array("value")),
("positiveModulo", Array("toYYYYMM(create_date)", "10"))
("modulo", Array("toYYYYMM(create_date)", "10"))
).foreach {
case (func_name: String, func_args: Array[String]) =>
test(s"shard by $func_name(${func_args.mkString(",")})")(runTest(func_name, func_args))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -186,5 +186,5 @@ object ExprUtils extends SQLConfHelper with Serializable {
}

def toSplitWithModulo(shardingKey: Expr, weight: Int): FuncExpr =
FuncExpr("positiveModulo", List(shardingKey, StringLiteral(weight.toString)))
FuncExpr("modulo", List(shardingKey, StringLiteral(weight.toString)))
}
Original file line number Diff line number Diff line change
Expand Up @@ -59,7 +59,7 @@ object StaticFunctionRegistry extends FunctionRegistry {
"clickhouse_months" -> Months,
"clickhouse_days" -> Days,
"clickhouse_hours" -> Hours,
"sharding_pmod" -> Pmod
"sharding_mod" -> Mod
)

override def list: Array[String] = functions.keys.toArray
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,15 +18,15 @@ import org.apache.spark.sql.connector.catalog.functions.{BoundFunction, ScalarFu
import org.apache.spark.sql.types._
import xenon.clickhouse.func.ClickhouseEquivFunction

object Pmod extends UnboundFunction with ScalarFunction[Long] with ClickhouseEquivFunction {
object Mod extends UnboundFunction with ScalarFunction[Long] with ClickhouseEquivFunction {

override def name: String = "sharding_pmod"
override def name: String = "sharding_mod"

override def canonicalName: String = s"clickhouse.$name"

override def toString: String = name

override val ckFuncNames: Array[String] = Array("positiveModulo", "positive_modulo", "pmod")
override val ckFuncNames: Array[String] = Array("modulo", "remainder")

override def description: String = s"$name: (a: long, b: long) => mod: long"

Expand Down Expand Up @@ -58,8 +58,5 @@ object Pmod extends UnboundFunction with ScalarFunction[Long] with ClickhouseEqu

override def isResultNullable: Boolean = false

def invoke(a: Long, b: Long): Long = {
val mod = a % b
if (mod < 0) mod + b else mod
}
def invoke(a: Long, b: Long): Long = a % b
}

0 comments on commit d9313e6

Please sign in to comment.