From 3eb53bd59e828275471d41730e6de601a887416d Mon Sep 17 00:00:00 2001 From: Cheng Hao Date: Wed, 7 May 2014 03:37:12 -0400 Subject: [WIP][Spark-SQL] Optimize the Constant Folding for Expression Currently, expression does not support the "constant null" well in constant folding. e.g. Sum(a, 0) actually always produces Literal(0, NumericType) in runtime. For example: ``` explain select isnull(key+null) from src; == Logical Plan == Project [HiveGenericUdf#isnull((key#30 + CAST(null, IntegerType))) AS c_0#28] MetastoreRelation default, src, None == Optimized Logical Plan == Project [true AS c_0#28] MetastoreRelation default, src, None == Physical Plan == Project [true AS c_0#28] HiveTableScan [], (MetastoreRelation default, src, None), None ``` I've create a new Optimization rule called NullPropagation for such kind of constant folding. Author: Cheng Hao Author: Michael Armbrust Closes #482 from chenghao-intel/optimize_constant_folding and squashes the following commits: 2f14b50 [Cheng Hao] Fix code style issues 68b9fad [Cheng Hao] Remove the Literal pattern matching for NullPropagation 29c8166 [Cheng Hao] Update the code for feedback of code review 50444cc [Cheng Hao] Remove the unnecessary null checking 80f9f18 [Cheng Hao] Update the UnitTest for aggregation constant folding 27ea3d7 [Cheng Hao] Fix Constant Folding Bugs & Add More Unittests b28e03a [Cheng Hao] Merge pull request #1 from marmbrus/pr/482 9ccefdb [Michael Armbrust] Add tests for optimized expression evaluation. 543ef9d [Cheng Hao] fix code style issues 9cf0396 [Cheng Hao] update code according to the code review comment 536c005 [Cheng Hao] Add Exceptional case for constant folding 3c045c7 [Cheng Hao] Optimize the Constant Folding by adding more rules 2645d4f [Cheng Hao] Constant Folding(null propagation) --- .../apache/spark/sql/catalyst/dsl/package.scala | 22 +- .../sql/catalyst/expressions/Expression.scala | 1 - .../spark/sql/catalyst/expressions/SortOrder.scala | 6 +- .../sql/catalyst/expressions/complexTypes.scala | 34 ++- .../sql/catalyst/expressions/predicates.scala | 3 +- .../spark/sql/catalyst/optimizer/Optimizer.scala | 67 +++++ .../expressions/ExpressionEvaluationSuite.scala | 115 +++++++- .../optimizer/ExpressionOptimizationSuite.scala | 36 +++ .../scala/org/apache/spark/sql/hive/hiveUdfs.scala | 11 + ...VG_SUM_COUNT-0-45f5619d9e4510195fe67f7c8d14a5c0 | 309 +++++++++++++++++++++ ...VG_SUM_COUNT-0-a393cfc24ad74f930f3284743254c10c | 309 +++++++++++++++++++++ ...VG_SUM_COUNT-0-ae497f1556f548c1e2da9244397a985d | 309 +++++++++++++++++++++ ...VG_SUM_COUNT-0-c60b6075da793b826db5eb4b08d7bab9 | 309 +++++++++++++++++++++ .../spark/sql/hive/execution/HiveQuerySuite.scala | 3 + 14 files changed, 1502 insertions(+), 32 deletions(-) create mode 100644 sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/optimizer/ExpressionOptimizationSuite.scala create mode 100644 sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-45f5619d9e4510195fe67f7c8d14a5c0 create mode 100644 sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-a393cfc24ad74f930f3284743254c10c create mode 100644 sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-ae497f1556f548c1e2da9244397a985d create mode 100644 sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-c60b6075da793b826db5eb4b08d7bab9 (limited to 'sql') diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/dsl/package.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/dsl/package.scala index 987befe8e2..dc83485df1 100644 --- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/dsl/package.scala +++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/dsl/package.scala @@ -114,37 +114,37 @@ package object dsl { def attr = analysis.UnresolvedAttribute(s) /** Creates a new AttributeReference of type boolean */ - def boolean = AttributeReference(s, BooleanType, nullable = false)() + def boolean = AttributeReference(s, BooleanType, nullable = true)() /** Creates a new AttributeReference of type byte */ - def byte = AttributeReference(s, ByteType, nullable = false)() + def byte = AttributeReference(s, ByteType, nullable = true)() /** Creates a new AttributeReference of type short */ - def short = AttributeReference(s, ShortType, nullable = false)() + def short = AttributeReference(s, ShortType, nullable = true)() /** Creates a new AttributeReference of type int */ - def int = AttributeReference(s, IntegerType, nullable = false)() + def int = AttributeReference(s, IntegerType, nullable = true)() /** Creates a new AttributeReference of type long */ - def long = AttributeReference(s, LongType, nullable = false)() + def long = AttributeReference(s, LongType, nullable = true)() /** Creates a new AttributeReference of type float */ - def float = AttributeReference(s, FloatType, nullable = false)() + def float = AttributeReference(s, FloatType, nullable = true)() /** Creates a new AttributeReference of type double */ - def double = AttributeReference(s, DoubleType, nullable = false)() + def double = AttributeReference(s, DoubleType, nullable = true)() /** Creates a new AttributeReference of type string */ - def string = AttributeReference(s, StringType, nullable = false)() + def string = AttributeReference(s, StringType, nullable = true)() /** Creates a new AttributeReference of type decimal */ - def decimal = AttributeReference(s, DecimalType, nullable = false)() + def decimal = AttributeReference(s, DecimalType, nullable = true)() /** Creates a new AttributeReference of type timestamp */ - def timestamp = AttributeReference(s, TimestampType, nullable = false)() + def timestamp = AttributeReference(s, TimestampType, nullable = true)() /** Creates a new AttributeReference of type binary */ - def binary = AttributeReference(s, BinaryType, nullable = false)() + def binary = AttributeReference(s, BinaryType, nullable = true)() } implicit class DslAttribute(a: AttributeReference) { diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/Expression.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/Expression.scala index dd9332ada8..41398ff956 100644 --- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/Expression.scala +++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/Expression.scala @@ -44,7 +44,6 @@ abstract class Expression extends TreeNode[Expression] { * - A [[expressions.Cast Cast]] or [[expressions.UnaryMinus UnaryMinus]] is foldable if its * child is foldable. */ - // TODO: Supporting more foldable expressions. For example, deterministic Hive UDFs. def foldable: Boolean = false def nullable: Boolean def references: Set[Attribute] diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/SortOrder.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/SortOrder.scala index 08b2f11d20..d2b7685e73 100644 --- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/SortOrder.scala +++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/SortOrder.scala @@ -18,6 +18,7 @@ package org.apache.spark.sql.catalyst.expressions import org.apache.spark.sql.catalyst.errors.TreeNodeException +import org.apache.spark.sql.catalyst.trees abstract sealed class SortDirection case object Ascending extends SortDirection @@ -27,7 +28,10 @@ case object Descending extends SortDirection * An expression that can be used to sort a tuple. This class extends expression primarily so that * transformations over expression will descend into its child. */ -case class SortOrder(child: Expression, direction: SortDirection) extends UnaryExpression { +case class SortOrder(child: Expression, direction: SortDirection) extends Expression + with trees.UnaryNode[Expression] { + + override def references = child.references override def dataType = child.dataType override def nullable = child.nullable diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/complexTypes.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/complexTypes.scala index c947155cb7..195ca2eb3d 100644 --- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/complexTypes.scala +++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/complexTypes.scala @@ -28,6 +28,7 @@ case class GetItem(child: Expression, ordinal: Expression) extends Expression { val children = child :: ordinal :: Nil /** `Null` is returned for invalid ordinals. */ override def nullable = true + override def foldable = child.foldable && ordinal.foldable override def references = children.flatMap(_.references).toSet def dataType = child.dataType match { case ArrayType(dt) => dt @@ -40,23 +41,27 @@ case class GetItem(child: Expression, ordinal: Expression) extends Expression { override def toString = s"$child[$ordinal]" override def eval(input: Row): Any = { - if (child.dataType.isInstanceOf[ArrayType]) { - val baseValue = child.eval(input).asInstanceOf[Seq[_]] - val o = ordinal.eval(input).asInstanceOf[Int] - if (baseValue == null) { - null - } else if (o >= baseValue.size || o < 0) { - null - } else { - baseValue(o) - } + val value = child.eval(input) + if (value == null) { + null } else { - val baseValue = child.eval(input).asInstanceOf[Map[Any, _]] val key = ordinal.eval(input) - if (baseValue == null) { + if (key == null) { null } else { - baseValue.get(key).orNull + if (child.dataType.isInstanceOf[ArrayType]) { + val baseValue = value.asInstanceOf[Seq[_]] + val o = key.asInstanceOf[Int] + if (o >= baseValue.size || o < 0) { + null + } else { + baseValue(o) + } + } else { + val baseValue = value.asInstanceOf[Map[Any, _]] + val key = ordinal.eval(input) + baseValue.get(key).orNull + } } } } @@ -69,7 +74,8 @@ case class GetField(child: Expression, fieldName: String) extends UnaryExpressio type EvaluatedType = Any def dataType = field.dataType - def nullable = field.nullable + override def nullable = field.nullable + override def foldable = child.foldable protected def structType = child.dataType match { case s: StructType => s diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/predicates.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/predicates.scala index 82c7af6844..6ee479939d 100644 --- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/predicates.scala +++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/predicates.scala @@ -65,8 +65,7 @@ abstract class BinaryPredicate extends BinaryExpression with Predicate { def nullable = left.nullable || right.nullable } -case class Not(child: Expression) extends Predicate with trees.UnaryNode[Expression] { - def references = child.references +case class Not(child: Expression) extends UnaryExpression with Predicate { override def foldable = child.foldable def nullable = child.nullable override def toString = s"NOT $child" diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/Optimizer.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/Optimizer.scala index c0a09a16ac..3037d45cc6 100644 --- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/Optimizer.scala +++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/Optimizer.scala @@ -26,6 +26,7 @@ import org.apache.spark.sql.catalyst.types._ object Optimizer extends RuleExecutor[LogicalPlan] { val batches = Batch("ConstantFolding", Once, + NullPropagation, ConstantFolding, BooleanSimplification, SimplifyFilters, @@ -85,6 +86,72 @@ object ColumnPruning extends Rule[LogicalPlan] { } } +/** + * Replaces [[catalyst.expressions.Expression Expressions]] that can be statically evaluated with + * equivalent [[catalyst.expressions.Literal Literal]] values. This rule is more specific with + * Null value propagation from bottom to top of the expression tree. + */ +object NullPropagation extends Rule[LogicalPlan] { + def apply(plan: LogicalPlan): LogicalPlan = plan transform { + case q: LogicalPlan => q transformExpressionsUp { + case e @ Count(Literal(null, _)) => Literal(0, e.dataType) + case e @ Sum(Literal(c, _)) if c == 0 => Literal(0, e.dataType) + case e @ Average(Literal(c, _)) if c == 0 => Literal(0.0, e.dataType) + case e @ IsNull(c) if c.nullable == false => Literal(false, BooleanType) + case e @ IsNotNull(c) if c.nullable == false => Literal(true, BooleanType) + case e @ GetItem(Literal(null, _), _) => Literal(null, e.dataType) + case e @ GetItem(_, Literal(null, _)) => Literal(null, e.dataType) + case e @ GetField(Literal(null, _), _) => Literal(null, e.dataType) + case e @ Coalesce(children) => { + val newChildren = children.filter(c => c match { + case Literal(null, _) => false + case _ => true + }) + if (newChildren.length == 0) { + Literal(null, e.dataType) + } else if (newChildren.length == 1) { + newChildren(0) + } else { + Coalesce(newChildren) + } + } + case e @ If(Literal(v, _), trueValue, falseValue) => if (v == true) trueValue else falseValue + case e @ In(Literal(v, _), list) if (list.exists(c => c match { + case Literal(candidate, _) if candidate == v => true + case _ => false + })) => Literal(true, BooleanType) + case e: UnaryMinus => e.child match { + case Literal(null, _) => Literal(null, e.dataType) + case _ => e + } + case e: Cast => e.child match { + case Literal(null, _) => Literal(null, e.dataType) + case _ => e + } + case e: Not => e.child match { + case Literal(null, _) => Literal(null, e.dataType) + case _ => e + } + // Put exceptional cases above if any + case e: BinaryArithmetic => e.children match { + case Literal(null, _) :: right :: Nil => Literal(null, e.dataType) + case left :: Literal(null, _) :: Nil => Literal(null, e.dataType) + case _ => e + } + case e: BinaryComparison => e.children match { + case Literal(null, _) :: right :: Nil => Literal(null, e.dataType) + case left :: Literal(null, _) :: Nil => Literal(null, e.dataType) + case _ => e + } + case e: StringRegexExpression => e.children match { + case Literal(null, _) :: right :: Nil => Literal(null, e.dataType) + case left :: Literal(null, _) :: Nil => Literal(null, e.dataType) + case _ => e + } + } + } +} + /** * Replaces [[catalyst.expressions.Expression Expressions]] that can be statically evaluated with * equivalent [[catalyst.expressions.Literal Literal]] values. diff --git a/sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/expressions/ExpressionEvaluationSuite.scala b/sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/expressions/ExpressionEvaluationSuite.scala index d287ad73b9..91605d0a26 100644 --- a/sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/expressions/ExpressionEvaluationSuite.scala +++ b/sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/expressions/ExpressionEvaluationSuite.scala @@ -108,9 +108,7 @@ class ExpressionEvaluationSuite extends FunSuite { truthTable.foreach { case (l,r,answer) => val expr = op(Literal(l, BooleanType), Literal(r, BooleanType)) - val result = expr.eval(null) - if (result != answer) - fail(s"$expr should not evaluate to $result, expected: $answer") + checkEvaluation(expr, answer) } } } @@ -131,6 +129,7 @@ class ExpressionEvaluationSuite extends FunSuite { test("LIKE literal Regular Expression") { checkEvaluation(Literal(null, StringType).like("a"), null) + checkEvaluation(Literal("a", StringType).like(Literal(null, StringType)), null) checkEvaluation(Literal(null, StringType).like(Literal(null, StringType)), null) checkEvaluation("abdef" like "abdef", true) checkEvaluation("a_%b" like "a\\__b", true) @@ -159,9 +158,14 @@ class ExpressionEvaluationSuite extends FunSuite { checkEvaluation("abc" like regEx, true, new GenericRow(Array[Any]("a%"))) checkEvaluation("abc" like regEx, false, new GenericRow(Array[Any]("b%"))) checkEvaluation("abc" like regEx, false, new GenericRow(Array[Any]("bc%"))) + + checkEvaluation(Literal(null, StringType) like regEx, null, new GenericRow(Array[Any]("bc%"))) } test("RLIKE literal Regular Expression") { + checkEvaluation(Literal(null, StringType) rlike "abdef", null) + checkEvaluation("abdef" rlike Literal(null, StringType), null) + checkEvaluation(Literal(null, StringType) rlike Literal(null, StringType), null) checkEvaluation("abdef" rlike "abdef", true) checkEvaluation("abbbbc" rlike "a.*c", true) @@ -257,6 +261,8 @@ class ExpressionEvaluationSuite extends FunSuite { assert(("abcdef" cast DecimalType).nullable === true) assert(("abcdef" cast DoubleType).nullable === true) assert(("abcdef" cast FloatType).nullable === true) + + checkEvaluation(Cast(Literal(null, IntegerType), ShortType), null) } test("timestamp") { @@ -287,5 +293,108 @@ class ExpressionEvaluationSuite extends FunSuite { // A test for higher precision than millis checkEvaluation(Cast(Cast(0.00000001, TimestampType), DoubleType), 0.00000001) } + + test("null checking") { + val row = new GenericRow(Array[Any]("^Ba*n", null, true, null)) + val c1 = 'a.string.at(0) + val c2 = 'a.string.at(1) + val c3 = 'a.boolean.at(2) + val c4 = 'a.boolean.at(3) + + checkEvaluation(IsNull(c1), false, row) + checkEvaluation(IsNotNull(c1), true, row) + + checkEvaluation(IsNull(c2), true, row) + checkEvaluation(IsNotNull(c2), false, row) + + checkEvaluation(IsNull(Literal(1, ShortType)), false) + checkEvaluation(IsNotNull(Literal(1, ShortType)), true) + + checkEvaluation(IsNull(Literal(null, ShortType)), true) + checkEvaluation(IsNotNull(Literal(null, ShortType)), false) + + checkEvaluation(Coalesce(c1 :: c2 :: Nil), "^Ba*n", row) + checkEvaluation(Coalesce(Literal(null, StringType) :: Nil), null, row) + checkEvaluation(Coalesce(Literal(null, StringType) :: c1 :: c2 :: Nil), "^Ba*n", row) + + checkEvaluation(If(c3, Literal("a", StringType), Literal("b", StringType)), "a", row) + checkEvaluation(If(c3, c1, c2), "^Ba*n", row) + checkEvaluation(If(c4, c2, c1), "^Ba*n", row) + checkEvaluation(If(Literal(null, BooleanType), c2, c1), "^Ba*n", row) + checkEvaluation(If(Literal(true, BooleanType), c1, c2), "^Ba*n", row) + checkEvaluation(If(Literal(false, BooleanType), c2, c1), "^Ba*n", row) + checkEvaluation(If(Literal(false, BooleanType), + Literal("a", StringType), Literal("b", StringType)), "b", row) + + checkEvaluation(In(c1, c1 :: c2 :: Nil), true, row) + checkEvaluation(In(Literal("^Ba*n", StringType), + Literal("^Ba*n", StringType) :: Nil), true, row) + checkEvaluation(In(Literal("^Ba*n", StringType), + Literal("^Ba*n", StringType) :: c2 :: Nil), true, row) + } + + test("complex type") { + val row = new GenericRow(Array[Any]( + "^Ba*n", // 0 + null.asInstanceOf[String], // 1 + new GenericRow(Array[Any]("aa", "bb")), // 2 + Map("aa"->"bb"), // 3 + Seq("aa", "bb") // 4 + )) + + val typeS = StructType( + StructField("a", StringType, true) :: StructField("b", StringType, true) :: Nil + ) + val typeMap = MapType(StringType, StringType) + val typeArray = ArrayType(StringType) + + checkEvaluation(GetItem(BoundReference(3, AttributeReference("c", typeMap)()), + Literal("aa")), "bb", row) + checkEvaluation(GetItem(Literal(null, typeMap), Literal("aa")), null, row) + checkEvaluation(GetItem(Literal(null, typeMap), Literal(null, StringType)), null, row) + checkEvaluation(GetItem(BoundReference(3, AttributeReference("c", typeMap)()), + Literal(null, StringType)), null, row) + + checkEvaluation(GetItem(BoundReference(4, AttributeReference("c", typeArray)()), + Literal(1)), "bb", row) + checkEvaluation(GetItem(Literal(null, typeArray), Literal(1)), null, row) + checkEvaluation(GetItem(Literal(null, typeArray), Literal(null, IntegerType)), null, row) + checkEvaluation(GetItem(BoundReference(4, AttributeReference("c", typeArray)()), + Literal(null, IntegerType)), null, row) + + checkEvaluation(GetField(BoundReference(2, AttributeReference("c", typeS)()), "a"), "aa", row) + checkEvaluation(GetField(Literal(null, typeS), "a"), null, row) + } + + test("arithmetic") { + val row = new GenericRow(Array[Any](1, 2, 3, null)) + val c1 = 'a.int.at(0) + val c2 = 'a.int.at(1) + val c3 = 'a.int.at(2) + val c4 = 'a.int.at(3) + + checkEvaluation(UnaryMinus(c1), -1, row) + checkEvaluation(UnaryMinus(Literal(100, IntegerType)), -100) + + checkEvaluation(Add(c1, c4), null, row) + checkEvaluation(Add(c1, c2), 3, row) + checkEvaluation(Add(c1, Literal(null, IntegerType)), null, row) + checkEvaluation(Add(Literal(null, IntegerType), c2), null, row) + checkEvaluation(Add(Literal(null, IntegerType), Literal(null, IntegerType)), null, row) + } + + test("BinaryComparison") { + val row = new GenericRow(Array[Any](1, 2, 3, null)) + val c1 = 'a.int.at(0) + val c2 = 'a.int.at(1) + val c3 = 'a.int.at(2) + val c4 = 'a.int.at(3) + + checkEvaluation(LessThan(c1, c4), null, row) + checkEvaluation(LessThan(c1, c2), true, row) + checkEvaluation(LessThan(c1, Literal(null, IntegerType)), null, row) + checkEvaluation(LessThan(Literal(null, IntegerType), c2), null, row) + checkEvaluation(LessThan(Literal(null, IntegerType), Literal(null, IntegerType)), null, row) + } } diff --git a/sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/optimizer/ExpressionOptimizationSuite.scala b/sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/optimizer/ExpressionOptimizationSuite.scala new file mode 100644 index 0000000000..890d6289b9 --- /dev/null +++ b/sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/optimizer/ExpressionOptimizationSuite.scala @@ -0,0 +1,36 @@ +/* + * 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. + */ + +package org.apache.spark.sql.catalyst.optimizer + +import org.apache.spark.sql.catalyst.expressions._ +import org.apache.spark.sql.catalyst.plans.logical._ + +/** + * Overrides our expression evaluation tests and reruns them after optimization has occured. This + * is to ensure that constant folding and other optimizations do not break anything. + */ +class ExpressionOptimizationSuite extends ExpressionEvaluationSuite { + override def checkEvaluation( + expression: Expression, + expected: Any, + inputRow: Row = EmptyRow): Unit = { + val plan = Project(Alias(expression, s"Optimized($expression)")() :: Nil, NoRelation) + val optimizedPlan = Optimizer(plan) + super.checkEvaluation(optimizedPlan.expressions.head, expected, inputRow) + } +} \ No newline at end of file diff --git a/sql/hive/src/main/scala/org/apache/spark/sql/hive/hiveUdfs.scala b/sql/hive/src/main/scala/org/apache/spark/sql/hive/hiveUdfs.scala index c7de4ab6d3..d50e2c65b7 100644 --- a/sql/hive/src/main/scala/org/apache/spark/sql/hive/hiveUdfs.scala +++ b/sql/hive/src/main/scala/org/apache/spark/sql/hive/hiveUdfs.scala @@ -22,6 +22,7 @@ import scala.collection.mutable.ArrayBuffer import org.apache.hadoop.hive.common.`type`.HiveDecimal import org.apache.hadoop.hive.ql.exec.UDF import org.apache.hadoop.hive.ql.exec.{FunctionInfo, FunctionRegistry} +import org.apache.hadoop.hive.ql.udf.{UDFType => HiveUDFType} import org.apache.hadoop.hive.ql.udf.generic._ import org.apache.hadoop.hive.serde2.objectinspector._ import org.apache.hadoop.hive.serde2.objectinspector.primitive._ @@ -237,6 +238,16 @@ private[hive] case class HiveGenericUdf(name: String, children: Seq[Expression]) @transient protected lazy val returnInspector = function.initialize(argumentInspectors.toArray) + @transient + protected lazy val isUDFDeterministic = { + val udfType = function.getClass().getAnnotation(classOf[HiveUDFType]) + (udfType != null && udfType.deterministic()) + } + + override def foldable = { + isUDFDeterministic && children.foldLeft(true)((prev, n) => prev && n.foldable) + } + val dataType: DataType = inspectorToDataType(returnInspector) override def eval(input: Row): Any = { diff --git a/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-45f5619d9e4510195fe67f7c8d14a5c0 b/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-45f5619d9e4510195fe67f7c8d14a5c0 new file mode 100644 index 0000000000..7643569a2c --- /dev/null +++ b/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-45f5619d9e4510195fe67f7c8d14a5c0 @@ -0,0 +1,309 @@ +0.0 0 0 0 3 +2.0 0 2 0 1 +4.0 0 4 0 1 +5.0 0 15 0 3 +8.0 0 8 0 1 +9.0 0 9 0 1 +10.0 0 10 0 1 +11.0 0 11 0 1 +12.0 0 24 0 2 +15.0 0 30 0 2 +17.0 0 17 0 1 +18.0 0 36 0 2 +19.0 0 19 0 1 +20.0 0 20 0 1 +24.0 0 48 0 2 +26.0 0 52 0 2 +27.0 0 27 0 1 +28.0 0 28 0 1 +30.0 0 30 0 1 +33.0 0 33 0 1 +34.0 0 34 0 1 +35.0 0 105 0 3 +37.0 0 74 0 2 +41.0 0 41 0 1 +42.0 0 84 0 2 +43.0 0 43 0 1 +44.0 0 44 0 1 +47.0 0 47 0 1 +51.0 0 102 0 2 +53.0 0 53 0 1 +54.0 0 54 0 1 +57.0 0 57 0 1 +58.0 0 116 0 2 +64.0 0 64 0 1 +65.0 0 65 0 1 +66.0 0 66 0 1 +67.0 0 134 0 2 +69.0 0 69 0 1 +70.0 0 210 0 3 +72.0 0 144 0 2 +74.0 0 74 0 1 +76.0 0 152 0 2 +77.0 0 77 0 1 +78.0 0 78 0 1 +80.0 0 80 0 1 +82.0 0 82 0 1 +83.0 0 166 0 2 +84.0 0 168 0 2 +85.0 0 85 0 1 +86.0 0 86 0 1 +87.0 0 87 0 1 +90.0 0 270 0 3 +92.0 0 92 0 1 +95.0 0 190 0 2 +96.0 0 96 0 1 +97.0 0 194 0 2 +98.0 0 196 0 2 +100.0 0 200 0 2 +103.0 0 206 0 2 +104.0 0 208 0 2 +105.0 0 105 0 1 +111.0 0 111 0 1 +113.0 0 226 0 2 +114.0 0 114 0 1 +116.0 0 116 0 1 +118.0 0 236 0 2 +119.0 0 357 0 3 +120.0 0 240 0 2 +125.0 0 250 0 2 +126.0 0 126 0 1 +128.0 0 384 0 3 +129.0 0 258 0 2 +131.0 0 131 0 1 +133.0 0 133 0 1 +134.0 0 268 0 2 +136.0 0 136 0 1 +137.0 0 274 0 2 +138.0 0 552 0 4 +143.0 0 143 0 1 +145.0 0 145 0 1 +146.0 0 292 0 2 +149.0 0 298 0 2 +150.0 0 150 0 1 +152.0 0 304 0 2 +153.0 0 153 0 1 +155.0 0 155 0 1 +156.0 0 156 0 1 +157.0 0 157 0 1 +158.0 0 158 0 1 +160.0 0 160 0 1 +162.0 0 162 0 1 +163.0 0 163 0 1 +164.0 0 328 0 2 +165.0 0 330 0 2 +166.0 0 166 0 1 +167.0 0 501 0 3 +168.0 0 168 0 1 +169.0 0 676 0 4 +170.0 0 170 0 1 +172.0 0 344 0 2 +174.0 0 348 0 2 +175.0 0 350 0 2 +176.0 0 352 0 2 +177.0 0 177 0 1 +178.0 0 178 0 1 +179.0 0 358 0 2 +180.0 0 180 0 1 +181.0 0 181 0 1 +183.0 0 183 0 1 +186.0 0 186 0 1 +187.0 0 561 0 3 +189.0 0 189 0 1 +190.0 0 190 0 1 +191.0 0 382 0 2 +192.0 0 192 0 1 +193.0 0 579 0 3 +194.0 0 194 0 1 +195.0 0 390 0 2 +196.0 0 196 0 1 +197.0 0 394 0 2 +199.0 0 597 0 3 +200.0 0 400 0 2 +201.0 0 201 0 1 +202.0 0 202 0 1 +203.0 0 406 0 2 +205.0 0 410 0 2 +207.0 0 414 0 2 +208.0 0 624 0 3 +209.0 0 418 0 2 +213.0 0 426 0 2 +214.0 0 214 0 1 +216.0 0 432 0 2 +217.0 0 434 0 2 +218.0 0 218 0 1 +219.0 0 438 0 2 +221.0 0 442 0 2 +222.0 0 222 0 1 +223.0 0 446 0 2 +224.0 0 448 0 2 +226.0 0 226 0 1 +228.0 0 228 0 1 +229.0 0 458 0 2 +230.0 0 1150 0 5 +233.0 0 466 0 2 +235.0 0 235 0 1 +237.0 0 474 0 2 +238.0 0 476 0 2 +239.0 0 478 0 2 +241.0 0 241 0 1 +242.0 0 484 0 2 +244.0 0 244 0 1 +247.0 0 247 0 1 +248.0 0 248 0 1 +249.0 0 249 0 1 +252.0 0 252 0 1 +255.0 0 510 0 2 +256.0 0 512 0 2 +257.0 0 257 0 1 +258.0 0 258 0 1 +260.0 0 260 0 1 +262.0 0 262 0 1 +263.0 0 263 0 1 +265.0 0 530 0 2 +266.0 0 266 0 1 +272.0 0 544 0 2 +273.0 0 819 0 3 +274.0 0 274 0 1 +275.0 0 275 0 1 +277.0 0 1108 0 4 +278.0 0 556 0 2 +280.0 0 560 0 2 +281.0 0 562 0 2 +282.0 0 564 0 2 +283.0 0 283 0 1 +284.0 0 284 0 1 +285.0 0 285 0 1 +286.0 0 286 0 1 +287.0 0 287 0 1 +288.0 0 576 0 2 +289.0 0 289 0 1 +291.0 0 291 0 1 +292.0 0 292 0 1 +296.0 0 296 0 1 +298.0 0 894 0 3 +302.0 0 302 0 1 +305.0 0 305 0 1 +306.0 0 306 0 1 +307.0 0 614 0 2 +308.0 0 308 0 1 +309.0 0 618 0 2 +310.0 0 310 0 1 +311.0 0 933 0 3 +315.0 0 315 0 1 +316.0 0 948 0 3 +317.0 0 634 0 2 +318.0 0 954 0 3 +321.0 0 642 0 2 +322.0 0 644 0 2 +323.0 0 323 0 1 +325.0 0 650 0 2 +327.0 0 981 0 3 +331.0 0 662 0 2 +332.0 0 332 0 1 +333.0 0 666 0 2 +335.0 0 335 0 1 +336.0 0 336 0 1 +338.0 0 338 0 1 +339.0 0 339 0 1 +341.0 0 341 0 1 +342.0 0 684 0 2 +344.0 0 688 0 2 +345.0 0 345 0 1 +348.0 0 1740 0 5 +351.0 0 351 0 1 +353.0 0 706 0 2 +356.0 0 356 0 1 +360.0 0 360 0 1 +362.0 0 362 0 1 +364.0 0 364 0 1 +365.0 0 365 0 1 +366.0 0 366 0 1 +367.0 0 734 0 2 +368.0 0 368 0 1 +369.0 0 1107 0 3 +373.0 0 373 0 1 +374.0 0 374 0 1 +375.0 0 375 0 1 +377.0 0 377 0 1 +378.0 0 378 0 1 +379.0 0 379 0 1 +382.0 0 764 0 2 +384.0 0 1152 0 3 +386.0 0 386 0 1 +389.0 0 389 0 1 +392.0 0 392 0 1 +393.0 0 393 0 1 +394.0 0 394 0 1 +395.0 0 790 0 2 +396.0 0 1188 0 3 +397.0 0 794 0 2 +399.0 0 798 0 2 +400.0 0 400 0 1 +401.0 0 2005 0 5 +402.0 0 402 0 1 +403.0 0 1209 0 3 +404.0 0 808 0 2 +406.0 0 1624 0 4 +407.0 0 407 0 1 +409.0 0 1227 0 3 +411.0 0 411 0 1 +413.0 0 826 0 2 +414.0 0 828 0 2 +417.0 0 1251 0 3 +418.0 0 418 0 1 +419.0 0 419 0 1 +421.0 0 421 0 1 +424.0 0 848 0 2 +427.0 0 427 0 1 +429.0 0 858 0 2 +430.0 0 1290 0 3 +431.0 0 1293 0 3 +432.0 0 432 0 1 +435.0 0 435 0 1 +436.0 0 436 0 1 +437.0 0 437 0 1 +438.0 0 1314 0 3 +439.0 0 878 0 2 +443.0 0 443 0 1 +444.0 0 444 0 1 +446.0 0 446 0 1 +448.0 0 448 0 1 +449.0 0 449 0 1 +452.0 0 452 0 1 +453.0 0 453 0 1 +454.0 0 1362 0 3 +455.0 0 455 0 1 +457.0 0 457 0 1 +458.0 0 916 0 2 +459.0 0 918 0 2 +460.0 0 460 0 1 +462.0 0 924 0 2 +463.0 0 926 0 2 +466.0 0 1398 0 3 +467.0 0 467 0 1 +468.0 0 1872 0 4 +469.0 0 2345 0 5 +470.0 0 470 0 1 +472.0 0 472 0 1 +475.0 0 475 0 1 +477.0 0 477 0 1 +478.0 0 956 0 2 +479.0 0 479 0 1 +480.0 0 1440 0 3 +481.0 0 481 0 1 +482.0 0 482 0 1 +483.0 0 483 0 1 +484.0 0 484 0 1 +485.0 0 485 0 1 +487.0 0 487 0 1 +489.0 0 1956 0 4 +490.0 0 490 0 1 +491.0 0 491 0 1 +492.0 0 984 0 2 +493.0 0 493 0 1 +494.0 0 494 0 1 +495.0 0 495 0 1 +496.0 0 496 0 1 +497.0 0 497 0 1 +498.0 0 1494 0 3 diff --git a/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-a393cfc24ad74f930f3284743254c10c b/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-a393cfc24ad74f930f3284743254c10c new file mode 100644 index 0000000000..f23b45c32e --- /dev/null +++ b/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-a393cfc24ad74f930f3284743254c10c @@ -0,0 +1,309 @@ +0.0 0.0 0 0 0 3 +0.0 2.0 0 2 0 1 +0.0 4.0 0 4 0 1 +0.0 5.0 0 15 0 3 +0.0 8.0 0 8 0 1 +0.0 9.0 0 9 0 1 +0.0 10.0 0 10 0 1 +0.0 11.0 0 11 0 1 +0.0 12.0 0 24 0 2 +0.0 15.0 0 30 0 2 +0.0 17.0 0 17 0 1 +0.0 18.0 0 36 0 2 +0.0 19.0 0 19 0 1 +0.0 20.0 0 20 0 1 +0.0 24.0 0 48 0 2 +0.0 26.0 0 52 0 2 +0.0 27.0 0 27 0 1 +0.0 28.0 0 28 0 1 +0.0 30.0 0 30 0 1 +0.0 33.0 0 33 0 1 +0.0 34.0 0 34 0 1 +0.0 35.0 0 105 0 3 +0.0 37.0 0 74 0 2 +0.0 41.0 0 41 0 1 +0.0 42.0 0 84 0 2 +0.0 43.0 0 43 0 1 +0.0 44.0 0 44 0 1 +0.0 47.0 0 47 0 1 +0.0 51.0 0 102 0 2 +0.0 53.0 0 53 0 1 +0.0 54.0 0 54 0 1 +0.0 57.0 0 57 0 1 +0.0 58.0 0 116 0 2 +0.0 64.0 0 64 0 1 +0.0 65.0 0 65 0 1 +0.0 66.0 0 66 0 1 +0.0 67.0 0 134 0 2 +0.0 69.0 0 69 0 1 +0.0 70.0 0 210 0 3 +0.0 72.0 0 144 0 2 +0.0 74.0 0 74 0 1 +0.0 76.0 0 152 0 2 +0.0 77.0 0 77 0 1 +0.0 78.0 0 78 0 1 +0.0 80.0 0 80 0 1 +0.0 82.0 0 82 0 1 +0.0 83.0 0 166 0 2 +0.0 84.0 0 168 0 2 +0.0 85.0 0 85 0 1 +0.0 86.0 0 86 0 1 +0.0 87.0 0 87 0 1 +0.0 90.0 0 270 0 3 +0.0 92.0 0 92 0 1 +0.0 95.0 0 190 0 2 +0.0 96.0 0 96 0 1 +0.0 97.0 0 194 0 2 +0.0 98.0 0 196 0 2 +0.0 100.0 0 200 0 2 +0.0 103.0 0 206 0 2 +0.0 104.0 0 208 0 2 +0.0 105.0 0 105 0 1 +0.0 111.0 0 111 0 1 +0.0 113.0 0 226 0 2 +0.0 114.0 0 114 0 1 +0.0 116.0 0 116 0 1 +0.0 118.0 0 236 0 2 +0.0 119.0 0 357 0 3 +0.0 120.0 0 240 0 2 +0.0 125.0 0 250 0 2 +0.0 126.0 0 126 0 1 +0.0 128.0 0 384 0 3 +0.0 129.0 0 258 0 2 +0.0 131.0 0 131 0 1 +0.0 133.0 0 133 0 1 +0.0 134.0 0 268 0 2 +0.0 136.0 0 136 0 1 +0.0 137.0 0 274 0 2 +0.0 138.0 0 552 0 4 +0.0 143.0 0 143 0 1 +0.0 145.0 0 145 0 1 +0.0 146.0 0 292 0 2 +0.0 149.0 0 298 0 2 +0.0 150.0 0 150 0 1 +0.0 152.0 0 304 0 2 +0.0 153.0 0 153 0 1 +0.0 155.0 0 155 0 1 +0.0 156.0 0 156 0 1 +0.0 157.0 0 157 0 1 +0.0 158.0 0 158 0 1 +0.0 160.0 0 160 0 1 +0.0 162.0 0 162 0 1 +0.0 163.0 0 163 0 1 +0.0 164.0 0 328 0 2 +0.0 165.0 0 330 0 2 +0.0 166.0 0 166 0 1 +0.0 167.0 0 501 0 3 +0.0 168.0 0 168 0 1 +0.0 169.0 0 676 0 4 +0.0 170.0 0 170 0 1 +0.0 172.0 0 344 0 2 +0.0 174.0 0 348 0 2 +0.0 175.0 0 350 0 2 +0.0 176.0 0 352 0 2 +0.0 177.0 0 177 0 1 +0.0 178.0 0 178 0 1 +0.0 179.0 0 358 0 2 +0.0 180.0 0 180 0 1 +0.0 181.0 0 181 0 1 +0.0 183.0 0 183 0 1 +0.0 186.0 0 186 0 1 +0.0 187.0 0 561 0 3 +0.0 189.0 0 189 0 1 +0.0 190.0 0 190 0 1 +0.0 191.0 0 382 0 2 +0.0 192.0 0 192 0 1 +0.0 193.0 0 579 0 3 +0.0 194.0 0 194 0 1 +0.0 195.0 0 390 0 2 +0.0 196.0 0 196 0 1 +0.0 197.0 0 394 0 2 +0.0 199.0 0 597 0 3 +0.0 200.0 0 400 0 2 +0.0 201.0 0 201 0 1 +0.0 202.0 0 202 0 1 +0.0 203.0 0 406 0 2 +0.0 205.0 0 410 0 2 +0.0 207.0 0 414 0 2 +0.0 208.0 0 624 0 3 +0.0 209.0 0 418 0 2 +0.0 213.0 0 426 0 2 +0.0 214.0 0 214 0 1 +0.0 216.0 0 432 0 2 +0.0 217.0 0 434 0 2 +0.0 218.0 0 218 0 1 +0.0 219.0 0 438 0 2 +0.0 221.0 0 442 0 2 +0.0 222.0 0 222 0 1 +0.0 223.0 0 446 0 2 +0.0 224.0 0 448 0 2 +0.0 226.0 0 226 0 1 +0.0 228.0 0 228 0 1 +0.0 229.0 0 458 0 2 +0.0 230.0 0 1150 0 5 +0.0 233.0 0 466 0 2 +0.0 235.0 0 235 0 1 +0.0 237.0 0 474 0 2 +0.0 238.0 0 476 0 2 +0.0 239.0 0 478 0 2 +0.0 241.0 0 241 0 1 +0.0 242.0 0 484 0 2 +0.0 244.0 0 244 0 1 +0.0 247.0 0 247 0 1 +0.0 248.0 0 248 0 1 +0.0 249.0 0 249 0 1 +0.0 252.0 0 252 0 1 +0.0 255.0 0 510 0 2 +0.0 256.0 0 512 0 2 +0.0 257.0 0 257 0 1 +0.0 258.0 0 258 0 1 +0.0 260.0 0 260 0 1 +0.0 262.0 0 262 0 1 +0.0 263.0 0 263 0 1 +0.0 265.0 0 530 0 2 +0.0 266.0 0 266 0 1 +0.0 272.0 0 544 0 2 +0.0 273.0 0 819 0 3 +0.0 274.0 0 274 0 1 +0.0 275.0 0 275 0 1 +0.0 277.0 0 1108 0 4 +0.0 278.0 0 556 0 2 +0.0 280.0 0 560 0 2 +0.0 281.0 0 562 0 2 +0.0 282.0 0 564 0 2 +0.0 283.0 0 283 0 1 +0.0 284.0 0 284 0 1 +0.0 285.0 0 285 0 1 +0.0 286.0 0 286 0 1 +0.0 287.0 0 287 0 1 +0.0 288.0 0 576 0 2 +0.0 289.0 0 289 0 1 +0.0 291.0 0 291 0 1 +0.0 292.0 0 292 0 1 +0.0 296.0 0 296 0 1 +0.0 298.0 0 894 0 3 +0.0 302.0 0 302 0 1 +0.0 305.0 0 305 0 1 +0.0 306.0 0 306 0 1 +0.0 307.0 0 614 0 2 +0.0 308.0 0 308 0 1 +0.0 309.0 0 618 0 2 +0.0 310.0 0 310 0 1 +0.0 311.0 0 933 0 3 +0.0 315.0 0 315 0 1 +0.0 316.0 0 948 0 3 +0.0 317.0 0 634 0 2 +0.0 318.0 0 954 0 3 +0.0 321.0 0 642 0 2 +0.0 322.0 0 644 0 2 +0.0 323.0 0 323 0 1 +0.0 325.0 0 650 0 2 +0.0 327.0 0 981 0 3 +0.0 331.0 0 662 0 2 +0.0 332.0 0 332 0 1 +0.0 333.0 0 666 0 2 +0.0 335.0 0 335 0 1 +0.0 336.0 0 336 0 1 +0.0 338.0 0 338 0 1 +0.0 339.0 0 339 0 1 +0.0 341.0 0 341 0 1 +0.0 342.0 0 684 0 2 +0.0 344.0 0 688 0 2 +0.0 345.0 0 345 0 1 +0.0 348.0 0 1740 0 5 +0.0 351.0 0 351 0 1 +0.0 353.0 0 706 0 2 +0.0 356.0 0 356 0 1 +0.0 360.0 0 360 0 1 +0.0 362.0 0 362 0 1 +0.0 364.0 0 364 0 1 +0.0 365.0 0 365 0 1 +0.0 366.0 0 366 0 1 +0.0 367.0 0 734 0 2 +0.0 368.0 0 368 0 1 +0.0 369.0 0 1107 0 3 +0.0 373.0 0 373 0 1 +0.0 374.0 0 374 0 1 +0.0 375.0 0 375 0 1 +0.0 377.0 0 377 0 1 +0.0 378.0 0 378 0 1 +0.0 379.0 0 379 0 1 +0.0 382.0 0 764 0 2 +0.0 384.0 0 1152 0 3 +0.0 386.0 0 386 0 1 +0.0 389.0 0 389 0 1 +0.0 392.0 0 392 0 1 +0.0 393.0 0 393 0 1 +0.0 394.0 0 394 0 1 +0.0 395.0 0 790 0 2 +0.0 396.0 0 1188 0 3 +0.0 397.0 0 794 0 2 +0.0 399.0 0 798 0 2 +0.0 400.0 0 400 0 1 +0.0 401.0 0 2005 0 5 +0.0 402.0 0 402 0 1 +0.0 403.0 0 1209 0 3 +0.0 404.0 0 808 0 2 +0.0 406.0 0 1624 0 4 +0.0 407.0 0 407 0 1 +0.0 409.0 0 1227 0 3 +0.0 411.0 0 411 0 1 +0.0 413.0 0 826 0 2 +0.0 414.0 0 828 0 2 +0.0 417.0 0 1251 0 3 +0.0 418.0 0 418 0 1 +0.0 419.0 0 419 0 1 +0.0 421.0 0 421 0 1 +0.0 424.0 0 848 0 2 +0.0 427.0 0 427 0 1 +0.0 429.0 0 858 0 2 +0.0 430.0 0 1290 0 3 +0.0 431.0 0 1293 0 3 +0.0 432.0 0 432 0 1 +0.0 435.0 0 435 0 1 +0.0 436.0 0 436 0 1 +0.0 437.0 0 437 0 1 +0.0 438.0 0 1314 0 3 +0.0 439.0 0 878 0 2 +0.0 443.0 0 443 0 1 +0.0 444.0 0 444 0 1 +0.0 446.0 0 446 0 1 +0.0 448.0 0 448 0 1 +0.0 449.0 0 449 0 1 +0.0 452.0 0 452 0 1 +0.0 453.0 0 453 0 1 +0.0 454.0 0 1362 0 3 +0.0 455.0 0 455 0 1 +0.0 457.0 0 457 0 1 +0.0 458.0 0 916 0 2 +0.0 459.0 0 918 0 2 +0.0 460.0 0 460 0 1 +0.0 462.0 0 924 0 2 +0.0 463.0 0 926 0 2 +0.0 466.0 0 1398 0 3 +0.0 467.0 0 467 0 1 +0.0 468.0 0 1872 0 4 +0.0 469.0 0 2345 0 5 +0.0 470.0 0 470 0 1 +0.0 472.0 0 472 0 1 +0.0 475.0 0 475 0 1 +0.0 477.0 0 477 0 1 +0.0 478.0 0 956 0 2 +0.0 479.0 0 479 0 1 +0.0 480.0 0 1440 0 3 +0.0 481.0 0 481 0 1 +0.0 482.0 0 482 0 1 +0.0 483.0 0 483 0 1 +0.0 484.0 0 484 0 1 +0.0 485.0 0 485 0 1 +0.0 487.0 0 487 0 1 +0.0 489.0 0 1956 0 4 +0.0 490.0 0 490 0 1 +0.0 491.0 0 491 0 1 +0.0 492.0 0 984 0 2 +0.0 493.0 0 493 0 1 +0.0 494.0 0 494 0 1 +0.0 495.0 0 495 0 1 +0.0 496.0 0 496 0 1 +0.0 497.0 0 497 0 1 +0.0 498.0 0 1494 0 3 diff --git a/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-ae497f1556f548c1e2da9244397a985d b/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-ae497f1556f548c1e2da9244397a985d new file mode 100644 index 0000000000..7839d714c2 --- /dev/null +++ b/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-ae497f1556f548c1e2da9244397a985d @@ -0,0 +1,309 @@ +0 3 +0 1 +0 1 +0 3 +0 1 +0 1 +0 1 +0 1 +0 2 +0 2 +0 1 +0 2 +0 1 +0 1 +0 2 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 3 +0 2 +0 1 +0 2 +0 1 +0 1 +0 1 +0 2 +0 1 +0 1 +0 1 +0 2 +0 1 +0 1 +0 1 +0 2 +0 1 +0 3 +0 2 +0 1 +0 2 +0 1 +0 1 +0 1 +0 1 +0 2 +0 2 +0 1 +0 1 +0 1 +0 3 +0 1 +0 2 +0 1 +0 2 +0 2 +0 2 +0 2 +0 2 +0 1 +0 1 +0 2 +0 1 +0 1 +0 2 +0 3 +0 2 +0 2 +0 1 +0 3 +0 2 +0 1 +0 1 +0 2 +0 1 +0 2 +0 4 +0 1 +0 1 +0 2 +0 2 +0 1 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 1 +0 1 +0 1 +0 2 +0 2 +0 1 +0 3 +0 1 +0 4 +0 1 +0 2 +0 2 +0 2 +0 2 +0 1 +0 1 +0 2 +0 1 +0 1 +0 1 +0 1 +0 3 +0 1 +0 1 +0 2 +0 1 +0 3 +0 1 +0 2 +0 1 +0 2 +0 3 +0 2 +0 1 +0 1 +0 2 +0 2 +0 2 +0 3 +0 2 +0 2 +0 1 +0 2 +0 2 +0 1 +0 2 +0 2 +0 1 +0 2 +0 2 +0 1 +0 1 +0 2 +0 5 +0 2 +0 1 +0 2 +0 2 +0 2 +0 1 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 2 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 2 +0 1 +0 2 +0 3 +0 1 +0 1 +0 4 +0 2 +0 2 +0 2 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 2 +0 1 +0 1 +0 1 +0 1 +0 3 +0 1 +0 1 +0 1 +0 2 +0 1 +0 2 +0 1 +0 3 +0 1 +0 3 +0 2 +0 3 +0 2 +0 2 +0 1 +0 2 +0 3 +0 2 +0 1 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 2 +0 2 +0 1 +0 5 +0 1 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 1 +0 2 +0 1 +0 3 +0 1 +0 1 +0 1 +0 1 +0 1 +0 1 +0 2 +0 3 +0 1 +0 1 +0 1 +0 1 +0 1 +0 2 +0 3 +0 2 +0 2 +0 1 +0 5 +0 1 +0 3 +0 2 +0 4 +0 1 +0 3 +0 1 +0 2 +0 2 +0 3 +0 1 +0 1 +0 1 +0 2 +0 1 +0 2 +0 3 +0 3 +0 1 +0 1 +0 1 +0 1 +0 3 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 1 +0 1 +0 3 +0 1 +0 1 +0 2 +0 2 +0 1 +0 2 +0 2 +0 3 +0 1 +0 4 +0 5 +0 1 +0 1 +0 1 +0 1 +0 2 +0 1 +0 3 +0 1 +0 1 +0 1 +0 1 +0 1 +0 1 +0 4 +0 1 +0 1 +0 2 +0 1 +0 1 +0 1 +0 1 +0 1 +0 3 diff --git a/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-c60b6075da793b826db5eb4b08d7bab9 b/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-c60b6075da793b826db5eb4b08d7bab9 new file mode 100644 index 0000000000..cd6b14ac2a --- /dev/null +++ b/sql/hive/src/test/resources/golden/Constant Folding Optimization for AVG_SUM_COUNT-0-c60b6075da793b826db5eb4b08d7bab9 @@ -0,0 +1,309 @@ +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 4 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 4 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 5 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 4 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 5 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 5 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 4 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 2 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 4 +0.0 0 0 5 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 3 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 4 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 2 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 1 +0.0 0 0 3 diff --git a/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveQuerySuite.scala b/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveQuerySuite.scala index d224d2ee60..87a92d8338 100644 --- a/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveQuerySuite.scala +++ b/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveQuerySuite.scala @@ -33,6 +33,9 @@ class HiveQuerySuite extends HiveComparisonTest { hiveql("FROM src SELECT key").collect() } + createQueryTest("Constant Folding Optimization for AVG_SUM_COUNT", + "SELECT AVG(0), SUM(0), COUNT(null), COUNT(value) FROM src GROUP BY key") + createQueryTest("Simple Average", "SELECT AVG(key) FROM src") -- cgit v1.2.3