Skip to content

Commit cccbaf1

Browse files
committed
fix
1 parent c90227a commit cccbaf1

File tree

1 file changed

+30
-17
lines changed

1 file changed

+30
-17
lines changed

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/SimplifyConditionalsInPredicate.scala

Lines changed: 30 additions & 17 deletions
Original file line numberDiff line numberDiff line change
@@ -22,9 +22,27 @@ import org.apache.spark.sql.catalyst.expressions.Literal.{FalseLiteral, TrueLite
2222
import org.apache.spark.sql.catalyst.plans.logical._
2323
import org.apache.spark.sql.catalyst.rules.Rule
2424
import org.apache.spark.sql.types.BooleanType
25-
import org.apache.spark.util.Utils
26-
2725

26+
/**
27+
* A rule that converting conditional expressions to predicate expressions, if possible, in the
28+
* search condition of the WHERE/HAVING/ON(JOIN) clauses, which contain an implicit Boolean operator
29+
* "(search condition) = TRUE". After this converting, we can potentially push the filter down to
30+
* the data source.
31+
*
32+
* Supported cases are:
33+
* - IF(cond, trueVal, false) => AND(cond, trueVal)
34+
* - IF(cond, trueVal, true) => OR(NOT(cond), trueVal)
35+
* - IF(cond, false, falseVal) => AND(NOT(cond), elseVal)
36+
* - IF(cond, true, falseVal) => OR(cond, elseVal)
37+
* - CASE WHEN cond THEN trueVal ELSE false END => AND(cond, trueVal)
38+
* - CASE WHEN cond THEN trueVal END => AND(cond, trueVal)
39+
* - CASE WHEN cond THEN trueVal ELSE null END => AND(cond, trueVal)
40+
* - CASE WHEN cond THEN trueVal ELSE true END => OR(NOT(cond), trueVal)
41+
* - CASE WHEN cond THEN false ELSE elseVal END => AND(NOT(cond), elseVal)
42+
* - CASE WHEN cond THEN false END => AND(NOT(cond), false)
43+
* - CASE WHEN cond THEN true ELSE elseVal END => OR(cond, elseVal)
44+
* - CASE WHEN cond THEN true END => OR(cond, false)
45+
*/
2846
object SimplifyConditionalsInPredicate extends Rule[LogicalPlan] {
2947

3048
def apply(plan: LogicalPlan): LogicalPlan = plan transform {
@@ -35,31 +53,26 @@ object SimplifyConditionalsInPredicate extends Rule[LogicalPlan] {
3553
}
3654

3755
private def simplifyConditional(e: Expression): Expression = e match {
38-
case Literal(null, BooleanType) => FalseLiteral
3956
case And(left, right) => And(simplifyConditional(left), simplifyConditional(right))
4057
case Or(left, right) => Or(simplifyConditional(left), simplifyConditional(right))
41-
case If(cond, t, FalseLiteral) => And(cond, t)
42-
case If(cond, t, TrueLiteral) => Or(Not(cond), t)
43-
case If(cond, FalseLiteral, f) => And(Not(cond), f)
44-
case If(cond, TrueLiteral, f) => Or(cond, f)
58+
case If(cond, trueValue, FalseLiteral) => And(cond, trueValue)
59+
case If(cond, trueValue, TrueLiteral) => Or(Not(cond), trueValue)
60+
case If(cond, FalseLiteral, falseValue) => And(Not(cond), falseValue)
61+
case If(cond, TrueLiteral, falseValue) => Or(cond, falseValue)
4562
case CaseWhen(Seq((cond, trueValue)),
4663
Some(FalseLiteral) | Some(Literal(null, BooleanType)) | None) =>
4764
And(cond, trueValue)
4865
case CaseWhen(Seq((cond, trueValue)), Some(TrueLiteral)) =>
4966
Or(Not(cond), trueValue)
5067
case CaseWhen(Seq((cond, FalseLiteral)), elseValue) =>
51-
And(Not(cond), elseValue.getOrElse(Literal(null, BooleanType)))
68+
And(Not(cond), elseValue.getOrElse(FalseLiteral))
5269
case CaseWhen(Seq((cond, TrueLiteral)), elseValue) =>
53-
Or(cond, elseValue.getOrElse(Literal(null, BooleanType)))
70+
Or(cond, elseValue.getOrElse(FalseLiteral))
5471
case e if e.dataType == BooleanType => e
5572
case e =>
56-
val message = "Expected a Boolean type expression in simplifyConditional, " +
57-
s"but got the type `${e.dataType.catalogString}` in `${e.sql}`."
58-
if (Utils.isTesting) {
59-
throw new IllegalArgumentException(message)
60-
} else {
61-
logWarning(message)
62-
e
63-
}
73+
assert(e.dataType != BooleanType,
74+
"Expected a Boolean type expression in simplifyConditional, " +
75+
s"but got the type `${e.dataType.catalogString}` in `${e.sql}`.")
76+
e
6477
}
6578
}

0 commit comments

Comments
 (0)