You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@spark.apache.org by li...@apache.org on 2019/05/24 18:25:47 UTC

[spark] branch master updated: [SPARK-27824][SQL] Make rule EliminateResolvedHint idempotent

This is an automated email from the ASF dual-hosted git repository.

lixiao pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/spark.git


The following commit(s) were added to refs/heads/master by this push:
     new de13f70  [SPARK-27824][SQL] Make rule EliminateResolvedHint idempotent
de13f70 is described below

commit de13f70ce1b3d5c90dc9b4f185e50dea3f1508b7
Author: maryannxue <ma...@apache.org>
AuthorDate: Fri May 24 11:25:22 2019 -0700

    [SPARK-27824][SQL] Make rule EliminateResolvedHint idempotent
    
    ## What changes were proposed in this pull request?
    
    This fix prevents the rule EliminateResolvedHint from being applied again if it's already applied.
    
    ## How was this patch tested?
    
    Added new UT.
    
    Closes #24692 from maryannxue/eliminatehint-bug.
    
    Authored-by: maryannxue <ma...@apache.org>
    Signed-off-by: gatorsmile <ga...@gmail.com>
---
 .../catalyst/optimizer/EliminateResolvedHint.scala |  2 +-
 .../scala/org/apache/spark/sql/JoinHintSuite.scala | 23 ++++++++++++++++++++++
 2 files changed, 24 insertions(+), 1 deletion(-)

diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/EliminateResolvedHint.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/EliminateResolvedHint.scala
index aebd660..6419f9c 100644
--- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/EliminateResolvedHint.scala
+++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/optimizer/EliminateResolvedHint.scala
@@ -29,7 +29,7 @@ object EliminateResolvedHint extends Rule[LogicalPlan] {
   // is using transformUp rather than resolveOperators.
   def apply(plan: LogicalPlan): LogicalPlan = {
     val pulledUp = plan transformUp {
-      case j: Join =>
+      case j: Join if j.hint == JoinHint.NONE =>
         val (newLeft, leftHints) = extractHintsFromPlan(j.left)
         val (newRight, rightHints) = extractHintsFromPlan(j.right)
         val newJoinHint = JoinHint(mergeHints(leftHints), mergeHints(rightHints))
diff --git a/sql/core/src/test/scala/org/apache/spark/sql/JoinHintSuite.scala b/sql/core/src/test/scala/org/apache/spark/sql/JoinHintSuite.scala
index 9c2dc0c..534560b 100644
--- a/sql/core/src/test/scala/org/apache/spark/sql/JoinHintSuite.scala
+++ b/sql/core/src/test/scala/org/apache/spark/sql/JoinHintSuite.scala
@@ -22,8 +22,10 @@ import scala.collection.mutable.ArrayBuffer
 import org.apache.log4j.{AppenderSkeleton, Level}
 import org.apache.log4j.spi.LoggingEvent
 
+import org.apache.spark.sql.catalyst.optimizer.EliminateResolvedHint
 import org.apache.spark.sql.catalyst.plans.PlanTest
 import org.apache.spark.sql.catalyst.plans.logical._
+import org.apache.spark.sql.catalyst.rules.RuleExecutor
 import org.apache.spark.sql.execution.joins._
 import org.apache.spark.sql.internal.SQLConf
 import org.apache.spark.sql.test.SharedSQLContext
@@ -557,4 +559,25 @@ class JoinHintSuite extends PlanTest with SharedSQLContext {
       }
     }
   }
+
+  test("Verify that the EliminatedResolvedHint rule is idempotent") {
+    withTempView("t1", "t2") {
+      Seq((1, "4"), (2, "2")).toDF("key", "value").createTempView("t1")
+      Seq((1, "1"), (2, "12.3"), (2, "123")).toDF("key", "value").createTempView("t2")
+      val df = sql("SELECT /*+ broadcast(t2) */ * from t1 join t2 ON t1.key = t2.key")
+      val optimize = new RuleExecutor[LogicalPlan] {
+        val batches = Batch("EliminateResolvedHint", FixedPoint(10), EliminateResolvedHint) :: Nil
+      }
+      val optimized = optimize.execute(df.logicalPlan)
+      val expectedHints =
+        JoinHint(
+          None,
+          Some(HintInfo(strategy = Some(BROADCAST)))) :: Nil
+      val joinHints = optimized collect {
+        case Join(_, _, _, _, hint) => hint
+        case _: ResolvedHint => fail("ResolvedHint should not appear after optimize.")
+      }
+      assert(joinHints == expectedHints)
+    }
+  }
 }


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