Skip to content

Commit f9bc035

Browse files
viiryaLorenzoMartini
authored andcommitted
[SPARK-32056][SQL] Coalesce partitions for repartition by expressions when AQE is enabled
This patch proposes to coalesce partitions for repartition by expressions without specifying number of partitions, when AQE is enabled. When repartition by some partition expressions, users can specify number of partitions or not. If the number of partitions is specified, we should not coalesce partitions because it breaks user expectation. But if without specifying number of partitions, AQE should be able to coalesce partitions as other shuffling. Yes. After this change, if users don't specify the number of partitions when repartitioning data by expressions, AQE will coalesce partitions. Added unit test. Closes apache#28900 from viirya/SPARK-32056. Authored-by: Liang-Chi Hsieh <[email protected]> Signed-off-by: Wenchen Fan <[email protected]>
1 parent f099edb commit f9bc035

File tree

4 files changed

+119
-31
lines changed

4 files changed

+119
-31
lines changed

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/plans/logical/basicLogicalOperators.scala

Lines changed: 15 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -28,6 +28,7 @@ import org.apache.spark.sql.catalyst.plans._
2828
import org.apache.spark.sql.catalyst.plans.physical.{HashPartitioning, Partitioning, RangePartitioning, RoundRobinPartitioning}
2929
import org.apache.spark.sql.catalyst.util.truncatedString
3030
import org.apache.spark.sql.connector.catalog.Identifier
31+
import org.apache.spark.sql.internal.SQLConf
3132
import org.apache.spark.sql.types._
3233
import org.apache.spark.util.random.RandomSampler
3334

@@ -948,16 +949,18 @@ case class Repartition(numPartitions: Int, shuffle: Boolean, child: LogicalPlan)
948949
}
949950

950951
/**
951-
* This method repartitions data using [[Expression]]s into `numPartitions`, and receives
952+
* This method repartitions data using [[Expression]]s into `optNumPartitions`, and receives
952953
* information about the number of partitions during execution. Used when a specific ordering or
953954
* distribution is expected by the consumer of the query result. Use [[Repartition]] for RDD-like
954-
* `coalesce` and `repartition`.
955+
* `coalesce` and `repartition`. If no `optNumPartitions` is given, by default it partitions data
956+
* into `numShufflePartitions` defined in `SQLConf`, and could be coalesced by AQE.
955957
*/
956958
case class RepartitionByExpression(
957959
partitionExpressions: Seq[Expression],
958960
child: LogicalPlan,
959-
numPartitions: Int) extends RepartitionOperation {
961+
optNumPartitions: Option[Int]) extends RepartitionOperation {
960962

963+
val numPartitions = optNumPartitions.getOrElse(SQLConf.get.numShufflePartitions)
961964
require(numPartitions > 0, s"Number of partitions ($numPartitions) must be positive.")
962965

963966
val partitioning: Partitioning = {
@@ -985,6 +988,15 @@ case class RepartitionByExpression(
985988
override def shuffle: Boolean = true
986989
}
987990

991+
object RepartitionByExpression {
992+
def apply(
993+
partitionExpressions: Seq[Expression],
994+
child: LogicalPlan,
995+
numPartitions: Int): RepartitionByExpression = {
996+
RepartitionByExpression(partitionExpressions, child, Some(numPartitions))
997+
}
998+
}
999+
9881000
/**
9891001
* A relation with one row. This is used in "SELECT ..." without a from clause.
9901002
*/

sql/core/src/main/scala/org/apache/spark/sql/Dataset.scala

Lines changed: 33 additions & 21 deletions
Original file line numberDiff line numberDiff line change
@@ -2991,17 +2991,9 @@ class Dataset[T] private[sql](
29912991
Repartition(numPartitions, shuffle = true, logicalPlan)
29922992
}
29932993

2994-
/**
2995-
* Returns a new Dataset partitioned by the given partitioning expressions into
2996-
* `numPartitions`. The resulting Dataset is hash partitioned.
2997-
*
2998-
* This is the same operation as "DISTRIBUTE BY" in SQL (Hive QL).
2999-
*
3000-
* @group typedrel
3001-
* @since 2.0.0
3002-
*/
3003-
@scala.annotation.varargs
3004-
def repartition(numPartitions: Int, partitionExprs: Column*): Dataset[T] = {
2994+
private def repartitionByExpression(
2995+
numPartitions: Option[Int],
2996+
partitionExprs: Seq[Column]): Dataset[T] = {
30052997
// The underlying `LogicalPlan` operator special-cases all-`SortOrder` arguments.
30062998
// However, we don't want to complicate the semantics of this API method.
30072999
// Instead, let's give users a friendly error message, pointing them to the new method.
@@ -3015,6 +3007,20 @@ class Dataset[T] private[sql](
30153007
}
30163008
}
30173009

3010+
/**
3011+
* Returns a new Dataset partitioned by the given partitioning expressions into
3012+
* `numPartitions`. The resulting Dataset is hash partitioned.
3013+
*
3014+
* This is the same operation as "DISTRIBUTE BY" in SQL (Hive QL).
3015+
*
3016+
* @group typedrel
3017+
* @since 2.0.0
3018+
*/
3019+
@scala.annotation.varargs
3020+
def repartition(numPartitions: Int, partitionExprs: Column*): Dataset[T] = {
3021+
repartitionByExpression(Some(numPartitions), partitionExprs)
3022+
}
3023+
30183024
/**
30193025
* Returns a new Dataset partitioned by the given partitioning expressions, using
30203026
* `spark.sql.shuffle.partitions` as number of partitions.
@@ -3027,7 +3033,20 @@ class Dataset[T] private[sql](
30273033
*/
30283034
@scala.annotation.varargs
30293035
def repartition(partitionExprs: Column*): Dataset[T] = {
3030-
repartition(sparkSession.sessionState.conf.numShufflePartitions, partitionExprs: _*)
3036+
repartitionByExpression(None, partitionExprs)
3037+
}
3038+
3039+
private def repartitionByRange(
3040+
numPartitions: Option[Int],
3041+
partitionExprs: Seq[Column]): Dataset[T] = {
3042+
require(partitionExprs.nonEmpty, "At least one partition-by expression must be specified.")
3043+
val sortOrder: Seq[SortOrder] = partitionExprs.map(_.expr match {
3044+
case expr: SortOrder => expr
3045+
case expr: Expression => SortOrder(expr, Ascending)
3046+
})
3047+
withTypedPlan {
3048+
RepartitionByExpression(sortOrder, logicalPlan, numPartitions)
3049+
}
30313050
}
30323051

30333052
/**
@@ -3049,14 +3068,7 @@ class Dataset[T] private[sql](
30493068
*/
30503069
@scala.annotation.varargs
30513070
def repartitionByRange(numPartitions: Int, partitionExprs: Column*): Dataset[T] = {
3052-
require(partitionExprs.nonEmpty, "At least one partition-by expression must be specified.")
3053-
val sortOrder: Seq[SortOrder] = partitionExprs.map(_.expr match {
3054-
case expr: SortOrder => expr
3055-
case expr: Expression => SortOrder(expr, Ascending)
3056-
})
3057-
withTypedPlan {
3058-
RepartitionByExpression(sortOrder, logicalPlan, numPartitions)
3059-
}
3071+
repartitionByRange(Some(numPartitions), partitionExprs)
30603072
}
30613073

30623074
/**
@@ -3078,7 +3090,7 @@ class Dataset[T] private[sql](
30783090
*/
30793091
@scala.annotation.varargs
30803092
def repartitionByRange(partitionExprs: Column*): Dataset[T] = {
3081-
repartitionByRange(sparkSession.sessionState.conf.numShufflePartitions, partitionExprs: _*)
3093+
repartitionByRange(None, partitionExprs)
30823094
}
30833095

30843096
/**

sql/core/src/main/scala/org/apache/spark/sql/execution/SparkStrategies.scala

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -787,8 +787,9 @@ abstract class SparkStrategies extends QueryPlanner[SparkPlan] {
787787
case r: logical.Range =>
788788
execution.RangeExec(r) :: Nil
789789
case r: logical.RepartitionByExpression =>
790+
val canChangeNumParts = r.optNumPartitions.isEmpty
790791
exchange.ShuffleExchangeExec(
791-
r.partitioning, planLater(r.child), noUserSpecifiedNumPartition = false) :: Nil
792+
r.partitioning, planLater(r.child), canChangeNumParts) :: Nil
792793
case ExternalRDD(outputObjAttr, rdd) => ExternalRDDScanExec(outputObjAttr, rdd) :: Nil
793794
case r: LogicalRDD =>
794795
RDDScanExec(r.output, r.rdd, "ExistingRDD", r.outputPartitioning, r.outputOrdering) :: Nil

sql/core/src/test/scala/org/apache/spark/sql/execution/adaptive/AdaptiveQueryExecSuite.scala

Lines changed: 69 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -874,18 +874,81 @@ class AdaptiveQueryExecSuite
874874
}
875875
}
876876

877-
test("SPARK-31220 repartition obeys initialPartitionNum when adaptiveExecutionEnabled") {
877+
test("SPARK-31220, SPARK-32056: repartition by expression with AQE") {
878878
Seq(true, false).foreach { enableAQE =>
879879
withSQLConf(
880880
SQLConf.ADAPTIVE_EXECUTION_ENABLED.key -> enableAQE.toString,
881-
SQLConf.SHUFFLE_PARTITIONS.key -> "6",
882-
SQLConf.COALESCE_PARTITIONS_INITIAL_PARTITION_NUM.key -> "7") {
883-
val partitionsNum = spark.range(10).repartition($"id").rdd.collectPartitions().length
881+
SQLConf.COALESCE_PARTITIONS_ENABLED.key -> "true",
882+
SQLConf.COALESCE_PARTITIONS_INITIAL_PARTITION_NUM.key -> "10",
883+
SQLConf.SHUFFLE_PARTITIONS.key -> "10") {
884+
885+
val df1 = spark.range(10).repartition($"id")
886+
val df2 = spark.range(10).repartition($"id" + 1)
887+
888+
val partitionsNum1 = df1.rdd.collectPartitions().length
889+
val partitionsNum2 = df2.rdd.collectPartitions().length
890+
884891
if (enableAQE) {
885-
assert(partitionsNum === 7)
892+
assert(partitionsNum1 < 10)
893+
assert(partitionsNum2 < 10)
894+
895+
// repartition obeys initialPartitionNum when adaptiveExecutionEnabled
896+
val plan = df1.queryExecution.executedPlan
897+
assert(plan.isInstanceOf[AdaptiveSparkPlanExec])
898+
val shuffle = plan.asInstanceOf[AdaptiveSparkPlanExec].executedPlan.collect {
899+
case s: ShuffleExchangeExec => s
900+
}
901+
assert(shuffle.size == 1)
902+
assert(shuffle(0).outputPartitioning.numPartitions == 10)
886903
} else {
887-
assert(partitionsNum === 6)
904+
assert(partitionsNum1 === 10)
905+
assert(partitionsNum2 === 10)
888906
}
907+
908+
909+
// Don't coalesce partitions if the number of partitions is specified.
910+
val df3 = spark.range(10).repartition(10, $"id")
911+
val df4 = spark.range(10).repartition(10)
912+
assert(df3.rdd.collectPartitions().length == 10)
913+
assert(df4.rdd.collectPartitions().length == 10)
914+
}
915+
}
916+
}
917+
918+
test("SPARK-31220, SPARK-32056: repartition by range with AQE") {
919+
Seq(true, false).foreach { enableAQE =>
920+
withSQLConf(
921+
SQLConf.ADAPTIVE_EXECUTION_ENABLED.key -> enableAQE.toString,
922+
SQLConf.COALESCE_PARTITIONS_ENABLED.key -> "true",
923+
SQLConf.COALESCE_PARTITIONS_INITIAL_PARTITION_NUM.key -> "10",
924+
SQLConf.SHUFFLE_PARTITIONS.key -> "10") {
925+
926+
val df1 = spark.range(10).toDF.repartitionByRange($"id".asc)
927+
val df2 = spark.range(10).toDF.repartitionByRange(($"id" + 1).asc)
928+
929+
val partitionsNum1 = df1.rdd.collectPartitions().length
930+
val partitionsNum2 = df2.rdd.collectPartitions().length
931+
932+
if (enableAQE) {
933+
assert(partitionsNum1 < 10)
934+
assert(partitionsNum2 < 10)
935+
936+
// repartition obeys initialPartitionNum when adaptiveExecutionEnabled
937+
val plan = df1.queryExecution.executedPlan
938+
assert(plan.isInstanceOf[AdaptiveSparkPlanExec])
939+
val shuffle = plan.asInstanceOf[AdaptiveSparkPlanExec].executedPlan.collect {
940+
case s: ShuffleExchangeExec => s
941+
}
942+
assert(shuffle.size == 1)
943+
assert(shuffle(0).outputPartitioning.numPartitions == 10)
944+
} else {
945+
assert(partitionsNum1 === 10)
946+
assert(partitionsNum2 === 10)
947+
}
948+
949+
// Don't coalesce partitions if the number of partitions is specified.
950+
val df3 = spark.range(10).repartitionByRange(10, $"id".asc)
951+
assert(df3.rdd.collectPartitions().length == 10)
889952
}
890953
}
891954
}

0 commit comments

Comments
 (0)