Skip to content

Conversation

@advancedxy
Copy link
Contributor

What changes were proposed in this pull request?

A new optimizer strategy called PruneHiveTablePartitions is added, which calculates table size as the total size of pruned partitions. Thus, Spark planner can pick up BroadcastJoin if the size of pruned partitions is under broadcast join threshold.

Why are the changes needed?

This is a performance improvement.

Does this PR introduce any user-facing change?

No.

How was this patch tested?

Added unit tests.

This is based on #18193, credits should go to @lianhuiwang.

@advancedxy
Copy link
Contributor Author

cc @cloud-fan.

@cloud-fan
Copy link
Contributor

ok to test

@cloud-fan
Copy link
Contributor

add to whitelist

predicate.references.subsetOf(partitionSet)
}
val conf = session.sessionState.conf
if (pruningPredicates.nonEmpty && conf.fallBackToHdfsForStatsEnabled &&
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Why we need to check conf.fallBackToHdfsForStatsEnabled?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

We should only get size from HDFS if conf.fallBackToHdfsForStatsEnabled? Since it could be a time-consuming operation.

Though, this condition should probably be pushed down to before the CommandUtils.calculateLocationSize call

@SparkQA
Copy link

SparkQA commented Sep 24, 2019

Test build #111297 has finished for PR 25919 at commit ffb7168.

  • This patch fails Spark unit tests.
  • This patch merges cleanly.
  • This patch adds the following public classes (experimental):
  • case class PruneHiveTablePartitions(

@advancedxy advancedxy changed the title [SPARK-15616][SQL] Hive table supports partition pruning in JoinSelection [WIP][SPARK-15616][SQL] Hive table supports partition pruning in JoinSelection Oct 20, 2019
@advancedxy
Copy link
Contributor Author

@cloud-fan Now pruned partitions are cached in HiveTableRelation, what do you think about current approach ?

@SparkQA
Copy link

SparkQA commented Oct 20, 2019

Test build #112331 has finished for PR 25919 at commit c054d22.

  • This patch fails to build.
  • This patch does not merge cleanly.
  • This patch adds no public classes.

@SparkQA
Copy link

SparkQA commented Oct 20, 2019

Test build #112332 has finished for PR 25919 at commit e744da5.

  • This patch fails to build.
  • This patch merges cleanly.
  • This patch adds no public classes.

@SparkQA
Copy link

SparkQA commented Oct 20, 2019

Test build #112334 has finished for PR 25919 at commit 12e1dc5.

  • This patch fails to build.
  • This patch merges cleanly.
  • This patch adds no public classes.

@SparkQA
Copy link

SparkQA commented Oct 20, 2019

Test build #112337 has finished for PR 25919 at commit b334e99.

  • This patch fails Spark unit tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

@SparkQA
Copy link

SparkQA commented Oct 21, 2019

Test build #112356 has finished for PR 25919 at commit 8d615f7.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

tableStats: Option[Statistics] = None) extends LeafNode with MultiInstanceRelation {
tableStats: Option[Statistics] = None,
@transient normalizedFilters: Seq[Expression] = Nil,
@transient prunedPartitions: Seq[CatalogTablePartition] = Nil)
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

How can we distinguish 0 partitions after pruning, and not being partition pruned?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

We have another field called normalizedFilters, when it's empty(Nil), then the prunedPartitions are not pruned, otherwise it could be 0 partitions after pruning when prunedPartitions = Nil

case class PruneHiveTablePartitions(
session: SparkSession) extends Rule[LogicalPlan] with PredicateHelper {
override def apply(plan: LogicalPlan): LogicalPlan = plan resolveOperators {
case filter @ Filter(condition, relation: HiveTableRelation) if relation.isPartitioned =>
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

can we follow PruneFileSourcePartitions? I think we should also support Filter(Project(HiveScan))

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Will do.

normalizedFilters)
val isFiltersEqual = normalizedFilters.zip(relation.normalizedFilters)
.forall { case (e1, e2) => e1.semanticEquals(e2) }
if (isFiltersEqual) {
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

What are we doing here?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Only under exactly matched pruning filters, we can simply get partitions from HiveTableRelation

val withStats = relation.tableMeta.copy(
stats = Some(CatalogStatistics(sizeInBytes = BigInt(sizeInBytes))))
val prunedHiveTableRelation = relation.copy(tableMeta = withStats,
normalizedFilters = pruningPredicates, prunedPartitions = prunedPartitions)
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Why do we need to keep pruningPredicates? IIUC the approach should be very simply:

  1. this rule only changes HiveTableRelation to hold an optional partition list.
  2. the HiveTableScanExec will get the partition list from HiveTableRelation or call listPartitionsByFilter.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Due to SPARK-24085, the pruningPredicates(we eliminate the subquery) could be different than the filters passed to HiveTableScan. So I keep the pruningPredicates, and only retrieves the prunedPartitions when HiveTableScanExec's pruningPartitionPredict matches exactly with HiveTableRelation's normalizedFilters.

The simplified solution occurred to me first, then I thought the filters could be different for some reason, and SPARK-24085 is an example, hence the proposed solution here.

1. don't store pruningFilters in HiveTableRelation
2. follow PruneFilSourcePartitions's style to extract projections,
   predicates and hive relation
3. skip partition pruning if scalar subquery is involved.
@SparkQA
Copy link

SparkQA commented Oct 25, 2019

Test build #112679 has finished for PR 25919 at commit 86a0d9c.

  • This patch fails Scala style tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

@SparkQA
Copy link

SparkQA commented Oct 26, 2019

Test build #112707 has finished for PR 25919 at commit ecfbe4d.

  • This patch fails due to an unknown error code, -9.
  • This patch merges cleanly.
  • This patch adds no public classes.

@advancedxy
Copy link
Contributor Author

retest it please

@advancedxy
Copy link
Contributor Author

Gently ping @cloud-fan

@maropu
Copy link
Member

maropu commented Nov 24, 2019

retest this please

@maropu
Copy link
Member

maropu commented Nov 24, 2019

still WIP?

@SparkQA
Copy link

SparkQA commented Nov 24, 2019

Test build #114331 has finished for PR 25919 at commit ecfbe4d.

  • This patch passes all tests.
  • This patch merges cleanly.
  • This patch adds no public classes.

@advancedxy advancedxy changed the title [WIP][SPARK-15616][SQL] Hive table supports partition pruning in JoinSelection [SPARK-15616][SQL] Hive table supports partition pruning in JoinSelection Nov 25, 2019
@advancedxy
Copy link
Contributor Author

still WIP?

I think it's ready for review.

val normalizedFilters = partitionPruningPred.map(_.transform {
case a: AttributeReference => originalAttributes(a)
})
sparkSession.sessionState.catalog.listPartitionsByFilter(
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@cloud-fan @maropu @advancedxy
Since the rawPartitions are called by "prunePartitions(rawPartitions)" in doExecute method, it seems prunePartitions will filter out all irrelevant partitions using "boundPruningPred". Then why we still need to call listpartitionsByFilter here ?
Could you please help me understand this ? thanks a lot in advance.

!predicate.references.isEmpty && predicate.references.subsetOf(partitionSet)
}
// SPARK-24085: scalar subquery should be skipped for partition pruning
val hasScalarSubquery = pruningPredicates.exists(SubqueryExpression.hasSubquery)
Copy link
Contributor

@fuwhu fuwhu Dec 2, 2019

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

It skips all subqueries instead of scalar subqueries.

rawDataSize.get
} else if (totalSize.isDefined && totalSize.get > 0L) {
totalSize.get
} else if (conf.fallBackToHdfsForStatsEnabled) {
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Per the doc of the conf "spark.sql.statistics.fallBackToHdfs", it is only for non-partitioned hive table :
"This flag is effective only for non-partitioned Hive tables."

@advancedxy
Copy link
Contributor Author

closed in favor of #26805

@advancedxy advancedxy closed this Jan 8, 2020
cloud-fan pushed a commit that referenced this pull request Jan 21, 2020
### What changes were proposed in this pull request?
Add optimizer rule PruneHiveTablePartitions pruning hive table partitions based on filters on partition columns.
Doing so, the total size of pruned partitions may be small enough for broadcast join in JoinSelection strategy.

### Why are the changes needed?
In JoinSelection strategy, spark use the "plan.stats.sizeInBytes" to decide whether the plan is suitable for broadcast join.
Currently, "plan.stats.sizeInBytes" does not take "pruned partitions" into account, so it may miss some broadcast join and take sort-merge join instead, which will definitely impact join performance.
This PR aim at taking "pruned partitions" into account for hive table in "plan.stats.sizeInBytes" and then improve performance by using broadcast join if possible.

### Does this PR introduce any user-facing change?
no

### How was this patch tested?
Added unit tests.

This is based on #25919, credits should go to lianhuiwang and advancedxy.

Closes #26805 from fuwhu/SPARK-15616.

Authored-by: fuwhu <[email protected]>
Signed-off-by: Wenchen Fan <[email protected]>
fuwhu added a commit to fuwhu/spark that referenced this pull request Oct 26, 2020
Add optimizer rule PruneHiveTablePartitions pruning hive table partitions based on filters on partition columns.
Doing so, the total size of pruned partitions may be small enough for broadcast join in JoinSelection strategy.

In JoinSelection strategy, spark use the "plan.stats.sizeInBytes" to decide whether the plan is suitable for broadcast join.
Currently, "plan.stats.sizeInBytes" does not take "pruned partitions" into account, so it may miss some broadcast join and take sort-merge join instead, which will definitely impact join performance.
This PR aim at taking "pruned partitions" into account for hive table in "plan.stats.sizeInBytes" and then improve performance by using broadcast join if possible.

no

Added unit tests.

This is based on apache#25919, credits should go to lianhuiwang and advancedxy.

Closes apache#26805 from fuwhu/SPARK-15616.

Authored-by: fuwhu <[email protected]>
Signed-off-by: Wenchen Fan <[email protected]>
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Labels

Projects

None yet

Development

Successfully merging this pull request may close these issues.

6 participants