Skip to content

Commit

Permalink
[SPARK-12275][SQL] No plan for BroadcastHint in some condition
Browse files Browse the repository at this point in the history
When SparkStrategies.BasicOperators's "case BroadcastHint(child) => apply(child)" is hit, it only recursively invokes BasicOperators.apply with this "child". It makes many strategies have no change to process this plan, which probably leads to "No plan" issue, so we use planLater to go through all strategies.

https://issues.apache.org/jira/browse/SPARK-12275

Author: yucai <yucai.yu@intel.com>

Closes #10265 from yucai/broadcast_hint.
  • Loading branch information
yucai authored and yhuai committed Dec 14, 2015
1 parent 834e714 commit ed87f6d
Show file tree
Hide file tree
Showing 2 changed files with 8 additions and 1 deletion.
Original file line number Diff line number Diff line change
Expand Up @@ -364,7 +364,7 @@ private[sql] abstract class SparkStrategies extends QueryPlanner[SparkPlan] {
case e @ EvaluatePython(udf, child, _) =>
BatchPythonEvaluation(udf, e.output, planLater(child)) :: Nil
case LogicalRDD(output, rdd) => PhysicalRDD(output, rdd, "ExistingRDD") :: Nil
case BroadcastHint(child) => apply(child)
case BroadcastHint(child) => planLater(child) :: Nil
case _ => Nil
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -120,5 +120,12 @@ class DataFrameJoinSuite extends QueryTest with SharedSQLContext {

// planner should not crash without a join
broadcast(df1).queryExecution.executedPlan

// SPARK-12275: no physical plan for BroadcastHint in some condition
withTempPath { path =>
df1.write.parquet(path.getCanonicalPath)
val pf1 = sqlContext.read.parquet(path.getCanonicalPath)
assert(df1.join(broadcast(pf1)).count() === 4)
}
}
}

0 comments on commit ed87f6d

Please sign in to comment.