Skip to content

Commit 2c0fe81

Browse files
maropucloud-fan
authored andcommitted
[SPARK-22445][SQL][FOLLOW-UP] Respect stream-side child's needCopyResult in BroadcastHashJoin
## What changes were proposed in this pull request? I found #19656 causes some bugs, for example, it changed the result set of `q6` in tpcds (I keep tracking TPCDS results daily [here](https://github.com/maropu/spark-tpcds-datagen/tree/master/reports/tests)): - w/o pr19658 ``` +-----+---+ |state|cnt| +-----+---+ | MA| 10| | AK| 10| | AZ| 11| | ME| 13| | VT| 14| | NV| 15| | NH| 16| | UT| 17| | NJ| 21| | MD| 22| | WY| 25| | NM| 26| | OR| 31| | WA| 36| | ND| 38| | ID| 39| | SC| 45| | WV| 50| | FL| 51| | OK| 53| | MT| 53| | CO| 57| | AR| 58| | NY| 58| | PA| 62| | AL| 63| | LA| 63| | SD| 70| | WI| 80| | null| 81| | MI| 82| | NC| 82| | MS| 83| | CA| 84| | MN| 85| | MO| 88| | IL| 95| | IA|102| | TN|102| | IN|103| | KY|104| | NE|113| | OH|114| | VA|130| | KS|139| | GA|168| | TX|216| +-----+---+ ``` - w/ pr19658 ``` +-----+---+ |state|cnt| +-----+---+ | RI| 14| | AK| 16| | FL| 20| | NJ| 21| | NM| 21| | NV| 22| | MA| 22| | MD| 22| | UT| 22| | AZ| 25| | SC| 28| | AL| 36| | MT| 36| | WA| 39| | ND| 41| | MI| 44| | AR| 45| | OR| 47| | OK| 52| | PA| 53| | LA| 55| | CO| 55| | NY| 64| | WV| 66| | SD| 72| | MS| 73| | NC| 79| | IN| 82| | null| 85| | ID| 88| | MN| 91| | WI| 95| | IL| 96| | MO| 97| | CA|109| | CA|109| | TN|114| | NE|115| | KY|128| | OH|131| | IA|156| | TX|160| | VA|182| | KS|211| | GA|230| +-----+---+ ``` This pr is to keep the original logic of `CodegenContext.copyResult` in `BroadcastHashJoinExec`. ## How was this patch tested? Existing tests Author: Takeshi Yamamuro <[email protected]> Closes #19781 from maropu/SPARK-22445-bugfix.
1 parent e0d7665 commit 2c0fe81

File tree

2 files changed

+35
-7
lines changed

2 files changed

+35
-7
lines changed

sql/core/src/main/scala/org/apache/spark/sql/execution/joins/BroadcastHashJoinExec.scala

Lines changed: 9 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -76,20 +76,23 @@ case class BroadcastHashJoinExec(
7676
streamedPlan.asInstanceOf[CodegenSupport].inputRDDs()
7777
}
7878

79-
override def needCopyResult: Boolean = joinType match {
79+
private def multipleOutputForOneInput: Boolean = joinType match {
8080
case _: InnerLike | LeftOuter | RightOuter =>
8181
// For inner and outer joins, one row from the streamed side may produce multiple result rows,
82-
// if the build side has duplicated keys. Then we need to copy the result rows before putting
83-
// them in a buffer, because these result rows share one UnsafeRow instance. Note that here
84-
// we wait for the broadcast to be finished, which is a no-op because it's already finished
85-
// when we wait it in `doProduce`.
82+
// if the build side has duplicated keys. Note that here we wait for the broadcast to be
83+
// finished, which is a no-op because it's already finished when we wait it in `doProduce`.
8684
!buildPlan.executeBroadcast[HashedRelation]().value.keyIsUnique
8785

8886
// Other joins types(semi, anti, existence) can at most produce one result row for one input
89-
// row from the streamed side, so no need to copy the result rows.
87+
// row from the streamed side.
9088
case _ => false
9189
}
9290

91+
// If the streaming side needs to copy result, this join plan needs to copy too. Otherwise,
92+
// this join plan only needs to copy result if it may output multiple rows for one input.
93+
override def needCopyResult: Boolean =
94+
streamedPlan.asInstanceOf[CodegenSupport].needCopyResult || multipleOutputForOneInput
95+
9396
override def doProduce(ctx: CodegenContext): String = {
9497
streamedPlan.asInstanceOf[CodegenSupport].produce(ctx, this)
9598
}

sql/core/src/test/scala/org/apache/spark/sql/JoinSuite.scala

Lines changed: 26 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -25,7 +25,7 @@ import org.apache.spark.TestUtils.{assertNotSpilled, assertSpilled}
2525
import org.apache.spark.sql.catalyst.TableIdentifier
2626
import org.apache.spark.sql.catalyst.analysis.UnresolvedRelation
2727
import org.apache.spark.sql.catalyst.expressions.{Ascending, SortOrder}
28-
import org.apache.spark.sql.execution.SortExec
28+
import org.apache.spark.sql.execution.{BinaryExecNode, SortExec}
2929
import org.apache.spark.sql.execution.joins._
3030
import org.apache.spark.sql.internal.SQLConf
3131
import org.apache.spark.sql.test.SharedSQLContext
@@ -857,4 +857,29 @@ class JoinSuite extends QueryTest with SharedSQLContext {
857857

858858
joinQueries.foreach(assertJoinOrdering)
859859
}
860+
861+
test("SPARK-22445 Respect stream-side child's needCopyResult in BroadcastHashJoin") {
862+
val df1 = Seq((2, 3), (2, 5), (2, 2), (3, 8), (2, 1)).toDF("k", "v1")
863+
val df2 = Seq((2, 8), (3, 7), (3, 4), (1, 2)).toDF("k", "v2")
864+
val df3 = Seq((1, 1), (3, 2), (4, 3), (5, 1)).toDF("k", "v3")
865+
866+
withSQLConf(
867+
SQLConf.AUTO_BROADCASTJOIN_THRESHOLD.key -> "-1",
868+
SQLConf.JOIN_REORDER_ENABLED.key -> "false") {
869+
val df = df1.join(df2, "k").join(functions.broadcast(df3), "k")
870+
val plan = df.queryExecution.sparkPlan
871+
872+
// Check if `needCopyResult` in `BroadcastHashJoin` is correct when smj->bhj
873+
val joins = new collection.mutable.ArrayBuffer[BinaryExecNode]()
874+
plan.foreachUp {
875+
case j: BroadcastHashJoinExec => joins += j
876+
case j: SortMergeJoinExec => joins += j
877+
case _ =>
878+
}
879+
assert(joins.size == 2)
880+
assert(joins(0).isInstanceOf[SortMergeJoinExec])
881+
assert(joins(1).isInstanceOf[BroadcastHashJoinExec])
882+
checkAnswer(df, Row(3, 8, 7, 2) :: Row(3, 8, 4, 2) :: Nil)
883+
}
884+
}
860885
}

0 commit comments

Comments
 (0)