Skip to content

Commit

Permalink
[VL] support uuid function
Browse files Browse the repository at this point in the history
  • Loading branch information
zhli1142015 committed Mar 19, 2024
1 parent a1e0367 commit 4968b83
Show file tree
Hide file tree
Showing 7 changed files with 38 additions and 2 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -36,7 +36,7 @@ import org.apache.spark.sql.catalyst.{AggregateFunctionRewriteRule, FlushableHas
import org.apache.spark.sql.catalyst.analysis.FunctionRegistry.FunctionBuilder
import org.apache.spark.sql.catalyst.catalog.BucketSpec
import org.apache.spark.sql.catalyst.catalog.CatalogTypes.TablePartitionSpec
import org.apache.spark.sql.catalyst.expressions.{Alias, Ascending, Attribute, Cast, CreateNamedStruct, ElementAt, Expression, ExpressionInfo, Generator, GetArrayItem, GetMapValue, GetStructField, If, IsNaN, Literal, Murmur3Hash, NamedExpression, NaNvl, PosExplode, Round, SortOrder, StringSplit, StringTrim}
import org.apache.spark.sql.catalyst.expressions.{Alias, Ascending, Attribute, Cast, CreateNamedStruct, ElementAt, Expression, ExpressionInfo, Generator, GetArrayItem, GetMapValue, GetStructField, If, IsNaN, Literal, Murmur3Hash, NamedExpression, NaNvl, PosExplode, Round, SortOrder, StringSplit, StringTrim, Uuid}
import org.apache.spark.sql.catalyst.expressions.aggregate.{AggregateExpression, HLLAdapter}
import org.apache.spark.sql.catalyst.optimizer.BuildSide
import org.apache.spark.sql.catalyst.plans.JoinType
Expand Down Expand Up @@ -128,6 +128,13 @@ class SparkPlanExecApiImpl extends SparkPlanExecApi {
)
}

/** Transform Uuid to Substrait. */
override def genUuidTransformer(
substraitExprName: String,
original: Uuid): ExpressionTransformer = {
UuidTransformer(substraitExprName, original)
}

/** Transform map_entries to Substrait. */
override def genMapEntriesTransformer(
substraitExprName: String,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,7 +26,7 @@ import io.glutenproject.substrait.rel.LocalFilesNode.ReadFileFormat
import io.glutenproject.substrait.rel.LocalFilesNode.ReadFileFormat.{DwrfReadFormat, OrcReadFormat, ParquetReadFormat}

import org.apache.spark.sql.catalyst.catalog.BucketSpec
import org.apache.spark.sql.catalyst.expressions.{Alias, CumeDist, DenseRank, Descending, Expression, Lag, Lead, Literal, NamedExpression, NthValue, NTile, PercentRank, Rand, RangeFrame, Rank, RowNumber, SortOrder, SpecialFrameBoundary, SpecifiedWindowFrame}
import org.apache.spark.sql.catalyst.expressions.{Alias, CumeDist, DenseRank, Descending, Expression, Lag, Lead, Literal, NamedExpression, NthValue, NTile, PercentRank, Rand, RangeFrame, Rank, RowNumber, SortOrder, SpecialFrameBoundary, SpecifiedWindowFrame, Uuid}
import org.apache.spark.sql.catalyst.expressions.aggregate.{AggregateExpression, Count, Sum}
import org.apache.spark.sql.catalyst.plans.JoinType
import org.apache.spark.sql.catalyst.util.CharVarcharUtils
Expand Down Expand Up @@ -389,6 +389,7 @@ object BackendSettings extends BackendSettingsApi {
// Block directly falling back the below functions by FallbackEmptySchemaRelation.
case alias: Alias => checkExpr(alias.child)
case _: Rand => true
case _: Uuid => true
case _ => false
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -469,4 +469,10 @@ class VeloxFunctionsValidateSuite extends VeloxWholeStageTransformerSuite {
checkOperatorMatch[ProjectExecTransformer]
}
}

test("Test uuid function") {
runQueryAndCompare("""SELECT uuid() from lineitem limit 100""".stripMargin, false) {
checkOperatorMatch[ProjectExecTransformer]
}
}
}
1 change: 1 addition & 0 deletions docs/velox-backend-support-progress.md
Original file line number Diff line number Diff line change
Expand Up @@ -428,3 +428,4 @@ Gluten supports 199 functions. (Draw to right to see all data types)
| spark_partition_id | | | S | | | | | | | | | | | | | | | | | | | |
| stack | | | | | | | | | | | | | | | | | | | | | | |
| xxhash64 | xxhash64 | xxhash64 | | | | | | | | | | | | | | | | | | | | |
| uuid | uuid | uuid | S | | | | | | | | | | | | | | | | | | | |
Original file line number Diff line number Diff line change
Expand Up @@ -199,6 +199,10 @@ trait SparkPlanExecApi {
throw new UnsupportedOperationException("NaNvl is not supported")
}

def genUuidTransformer(substraitExprName: String, original: Uuid): ExpressionTransformer = {
GenericExpressionTransformer(substraitExprName, Seq(), original)
}

/** Transform map_entries to Substrait. */
def genMapEntriesTransformer(
substraitExprName: String,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -526,6 +526,8 @@ object ExpressionConverter extends SQLConfHelper with Logging {
val childrenTransformers =
e.children.map(replaceWithExpressionTransformerInternal(_, attributeSeq, expressionsMap))
e.getTransformer(childrenTransformers)
case u: Uuid =>
BackendsApiManager.getSparkPlanExecApiInstance.genUuidTransformer(substraitExprName, u)
case expr =>
GenericExpressionTransformer(
substraitExprName,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -239,3 +239,18 @@ case class GetArrayStructFieldsTransformer(
ExpressionBuilder.makeScalarFunction(functionId, inputNodes, typeNode)
}
}

case class UuidTransformer(substraitExprName: String, original: Uuid)
extends ExpressionTransformer {

override def doTransform(args: java.lang.Object): ExpressionNode = {
val functionMap = args.asInstanceOf[java.util.HashMap[String, java.lang.Long]]
val functionId = ExpressionBuilder.newScalarFunction(
functionMap,
ConverterUtils.makeFuncName(substraitExprName, Seq(LongType)))
val seed = Literal(original.randomSeed.get)
val inputNodes = Lists.newArrayList[ExpressionNode](LiteralTransformer(seed).doTransform(args))
val typeNode = ConverterUtils.getTypeNode(original.dataType, original.nullable)
ExpressionBuilder.makeScalarFunction(functionId, inputNodes, typeNode)
}
}

0 comments on commit 4968b83

Please sign in to comment.