Skip to content

Commit e12c9e7

Browse files
monkeyboy123cloud-fan
authored andcommitted
[SPARK-38333][SQL] PlanExpression expression should skip addExprTree function in Executor
It is master branch pr [SPARK-38333](#35662) Bug fix, it is potential issue. No UT Closes #36012 from monkeyboy123/spark-38333. Authored-by: Dereck Li <[email protected]> Signed-off-by: Wenchen Fan <[email protected]> (cherry picked from commit a40acd4) Signed-off-by: Wenchen Fan <[email protected]>
1 parent 4b3f8d4 commit e12c9e7

File tree

2 files changed

+18
-2
lines changed

2 files changed

+18
-2
lines changed

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/EquivalentExpressions.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -154,7 +154,7 @@ class EquivalentExpressions {
154154
expr.find(_.isInstanceOf[LambdaVariable]).isDefined ||
155155
// `PlanExpression` wraps query plan. To compare query plans of `PlanExpression` on executor,
156156
// can cause error like NPE.
157-
(expr.isInstanceOf[PlanExpression[_]] && TaskContext.get != null)
157+
(expr.find(_.isInstanceOf[PlanExpression[_]]).isDefined && TaskContext.get != null)
158158

159159
if (!skip && !addFunc(expr)) {
160160
childrenToRecurse(expr).foreach(addExprTree(_, addFunc))

sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/expressions/SubexpressionEliminationSuite.scala

Lines changed: 17 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -16,8 +16,9 @@
1616
*/
1717
package org.apache.spark.sql.catalyst.expressions
1818

19-
import org.apache.spark.SparkFunSuite
19+
import org.apache.spark.{SparkFunSuite, TaskContext, TaskContextImpl}
2020
import org.apache.spark.sql.catalyst.expressions.codegen._
21+
import org.apache.spark.sql.catalyst.plans.logical.LocalRelation
2122
import org.apache.spark.sql.internal.SQLConf
2223
import org.apache.spark.sql.types.{BinaryType, DataType, Decimal, IntegerType}
2324

@@ -326,6 +327,21 @@ class SubexpressionEliminationSuite extends SparkFunSuite with ExpressionEvalHel
326327
assert(equivalence.getAllEquivalentExprs.count(_.size == 2) == 0)
327328
}
328329

330+
test("SPARK-38333: PlanExpression expression should skip addExprTree function in Executor") {
331+
try {
332+
// suppose we are in executor
333+
val context1 = new TaskContextImpl(0, 0, 0, 0, 0, null, null, null, cpus = 0)
334+
TaskContext.setTaskContext(context1)
335+
336+
val equivalence = new EquivalentExpressions
337+
val expression = DynamicPruningExpression(Exists(LocalRelation()))
338+
equivalence.addExprTree(expression)
339+
assert(equivalence.getExprState(expression).isEmpty)
340+
} finally {
341+
TaskContext.unset()
342+
}
343+
}
344+
329345
test("SPARK-35886: PromotePrecision should not overwrite genCode") {
330346
val p = PromotePrecision(Literal(Decimal("10.1")))
331347

0 commit comments

Comments
 (0)