You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by GitBox <gi...@apache.org> on 2020/09/10 22:41:22 UTC

[GitHub] [spark] sunchao commented on a change in pull request #29661: [SPARK-32811][SQL] optimize IN predicate against continuous range

sunchao commented on a change in pull request #29661:
URL: https://github.com/apache/spark/pull/29661#discussion_r486670612



##########
File path: sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/expressions.scala
##########
@@ -231,10 +231,41 @@ object ReorderAssociativeOperator extends Rule[LogicalPlan] {
  * 1. Converts the predicate to false when the list is empty and
  *    the value is not nullable.
  * 2. Removes literal repetitions.
- * 3. Replaces [[In (value, seq[Literal])]] with optimized version
+ * 3. Replaces value IN (x,x+1,x+2..x+n) with x <= value AND value <= x + n
+ * 4. Replaces [[In (value, seq[Literal])]] with optimized version
  *    [[InSet (value, HashSet[Literal])]] which is much faster.
  */
 object OptimizeIn extends Rule[LogicalPlan] {
+  private def isContinousIntegers(nums: Set[Expression]): Boolean = {
+    if (nums.nonEmpty && isInteger(nums.head)) {
+      val (min, max) = getBound(nums)
+      val minL = min.eval(EmptyRow).asInstanceOf[Number].longValue()
+      val maxL = max.eval(EmptyRow).asInstanceOf[Number].longValue()
+       minL + (nums.size - 1) == maxL
+    } else {
+      false
+    }
+  }
+
+  private def isInteger(v: Expression): Boolean =

Review comment:
       Should we handle `ByteType` (tinyint) as well?

##########
File path: sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/expressions.scala
##########
@@ -231,10 +231,41 @@ object ReorderAssociativeOperator extends Rule[LogicalPlan] {
  * 1. Converts the predicate to false when the list is empty and
  *    the value is not nullable.
  * 2. Removes literal repetitions.
- * 3. Replaces [[In (value, seq[Literal])]] with optimized version
+ * 3. Replaces value IN (x,x+1,x+2..x+n) with x <= value AND value <= x + n
+ * 4. Replaces [[In (value, seq[Literal])]] with optimized version
  *    [[InSet (value, HashSet[Literal])]] which is much faster.
  */
 object OptimizeIn extends Rule[LogicalPlan] {
+  private def isContinousIntegers(nums: Set[Expression]): Boolean = {
+    if (nums.nonEmpty && isInteger(nums.head)) {
+      val (min, max) = getBound(nums)
+      val minL = min.eval(EmptyRow).asInstanceOf[Number].longValue()
+      val maxL = max.eval(EmptyRow).asInstanceOf[Number].longValue()
+       minL + (nums.size - 1) == maxL
+    } else {
+      false
+    }
+  }
+
+  private def isInteger(v: Expression): Boolean =
+    v.dataType.isInstanceOf[ShortType] ||
+      v.dataType.isInstanceOf[IntegerType] ||
+      v.dataType.isInstanceOf[LongType]
+
+  def getBound(nums: Set[Expression]): (Expression, Expression) = {

Review comment:
       make this private?

##########
File path: sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/expressions.scala
##########
@@ -231,10 +231,41 @@ object ReorderAssociativeOperator extends Rule[LogicalPlan] {
  * 1. Converts the predicate to false when the list is empty and
  *    the value is not nullable.
  * 2. Removes literal repetitions.
- * 3. Replaces [[In (value, seq[Literal])]] with optimized version
+ * 3. Replaces value IN (x,x+1,x+2..x+n) with x <= value AND value <= x + n
+ * 4. Replaces [[In (value, seq[Literal])]] with optimized version
  *    [[InSet (value, HashSet[Literal])]] which is much faster.
  */
 object OptimizeIn extends Rule[LogicalPlan] {
+  private def isContinousIntegers(nums: Set[Expression]): Boolean = {
+    if (nums.nonEmpty && isInteger(nums.head)) {
+      val (min, max) = getBound(nums)
+      val minL = min.eval(EmptyRow).asInstanceOf[Number].longValue()
+      val maxL = max.eval(EmptyRow).asInstanceOf[Number].longValue()
+       minL + (nums.size - 1) == maxL
+    } else {
+      false
+    }
+  }
+
+  private def isInteger(v: Expression): Boolean =
+    v.dataType.isInstanceOf[ShortType] ||
+      v.dataType.isInstanceOf[IntegerType] ||
+      v.dataType.isInstanceOf[LongType]
+
+  def getBound(nums: Set[Expression]): (Expression, Expression) = {
+    nums.head.dataType match {
+      case ShortType =>
+        val values = nums.map(e => e.eval(EmptyRow).asInstanceOf[Short])
+        (Literal(values.min, ShortType), Literal(values.max, ShortType))
+      case IntegerType =>
+        val values = nums.map(e => e.eval(EmptyRow).asInstanceOf[Integer])
+        (Literal(values.min, IntegerType), Literal(values.max, IntegerType))
+      case LongType =>
+        val values = nums.map(e => e.eval(EmptyRow).asInstanceOf[Long])
+        (Literal(values.min, LongType), Literal(values.max, LongType))
+    }

Review comment:
       perhaps we should add a default case to handle other types which are not short, int or long.

##########
File path: sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/expressions.scala
##########
@@ -231,10 +231,41 @@ object ReorderAssociativeOperator extends Rule[LogicalPlan] {
  * 1. Converts the predicate to false when the list is empty and
  *    the value is not nullable.
  * 2. Removes literal repetitions.
- * 3. Replaces [[In (value, seq[Literal])]] with optimized version
+ * 3. Replaces value IN (x,x+1,x+2..x+n) with x <= value AND value <= x + n
+ * 4. Replaces [[In (value, seq[Literal])]] with optimized version
  *    [[InSet (value, HashSet[Literal])]] which is much faster.
  */
 object OptimizeIn extends Rule[LogicalPlan] {
+  private def isContinousIntegers(nums: Set[Expression]): Boolean = {
+    if (nums.nonEmpty && isInteger(nums.head)) {
+      val (min, max) = getBound(nums)
+      val minL = min.eval(EmptyRow).asInstanceOf[Number].longValue()
+      val maxL = max.eval(EmptyRow).asInstanceOf[Number].longValue()
+       minL + (nums.size - 1) == maxL
+    } else {
+      false
+    }
+  }
+
+  private def isInteger(v: Expression): Boolean =
+    v.dataType.isInstanceOf[ShortType] ||
+      v.dataType.isInstanceOf[IntegerType] ||
+      v.dataType.isInstanceOf[LongType]
+
+  def getBound(nums: Set[Expression]): (Expression, Expression) = {
+    nums.head.dataType match {
+      case ShortType =>
+        val values = nums.map(e => e.eval(EmptyRow).asInstanceOf[Short])

Review comment:
       can `e.eval()` gets pretty expensive here? can `e` be any type of expressions other than literals?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org