Skip to content

Commit 0895d19

Browse files
mihailoale-dbcloud-fan
authored andcommitted
[SPARK-51614][SQL] Introduce ResolveUnresolvedHaving rule in the Analyzer
### What changes were proposed in this pull request? Introduce `ResolveUnresolvedHaving` rule in the Analyzer to handle `UnresolvedHaving` nodes with resolved `condition` and `child`. ### Why are the changes needed? Fix the case to work. ### Does this PR introduce _any_ user-facing change? Queries that failed now pass. ### How was this patch tested? Added test. ### Was this patch authored or co-authored using generative AI tooling? No. Closes #50409 from mihailoale-db/unresolvedhavingwithgenerate. Authored-by: mihailoale-db <[email protected]> Signed-off-by: Wenchen Fan <[email protected]>
1 parent c9fbcb1 commit 0895d19

File tree

3 files changed

+31
-1
lines changed

3 files changed

+31
-1
lines changed

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/Analyzer.scala

+17-1
Original file line numberDiff line numberDiff line change
@@ -460,7 +460,8 @@ class Analyzer(override val catalogManager: CatalogManager) extends RuleExecutor
460460
Batch("HandleSpecialCommand", Once,
461461
HandleSpecialCommand),
462462
Batch("Remove watermark for batch query", Once,
463-
EliminateEventTimeWatermark)
463+
EliminateEventTimeWatermark),
464+
Batch("ResolveUnresolvedHaving", Once, ResolveUnresolvedHaving)
464465
)
465466

466467
/**
@@ -4206,3 +4207,18 @@ object RemoveTempResolvedColumn extends Rule[LogicalPlan] {
42064207
}
42074208
}
42084209
}
4210+
4211+
/**
4212+
* Rule that's used to handle `UnresolvedHaving` nodes with resolved `condition` and `child`.
4213+
* It's placed outside the main batch to avoid conflicts with other rules that resolve
4214+
* `UnresolvedHaving` in the main batch.
4215+
*/
4216+
object ResolveUnresolvedHaving extends Rule[LogicalPlan] {
4217+
override def apply(plan: LogicalPlan): LogicalPlan = {
4218+
plan.resolveOperatorsWithPruning(_.containsPattern(UNRESOLVED_HAVING), ruleId) {
4219+
case u @ UnresolvedHaving(havingCondition, child)
4220+
if havingCondition.resolved && child.resolved =>
4221+
Filter(condition = havingCondition, child = child)
4222+
}
4223+
}
4224+
}

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/rules/RuleIdCollection.scala

+1
Original file line numberDiff line numberDiff line change
@@ -111,6 +111,7 @@ object RuleIdCollection {
111111
"org.apache.spark.sql.catalyst.analysis.ResolveUpdateEventTimeWatermarkColumn" ::
112112
"org.apache.spark.sql.catalyst.expressions.EliminatePipeOperators" ::
113113
"org.apache.spark.sql.catalyst.expressions.ValidateAndStripPipeExpressions" ::
114+
"org.apache.spark.sql.catalyst.analysis.ResolveUnresolvedHaving" ::
114115
// Catalyst Optimizer rules
115116
"org.apache.spark.sql.catalyst.optimizer.BooleanSimplification" ::
116117
"org.apache.spark.sql.catalyst.optimizer.CollapseProject" ::

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

+13
Original file line numberDiff line numberDiff line change
@@ -4943,6 +4943,19 @@ class SQLQuerySuite extends QueryTest with SharedSparkSession with AdaptiveSpark
49434943
Row(Array(0), Array(0)), Row(Array(1), Array(1)), Row(Array(2), Array(2)))
49444944
checkAnswer(df, expectedAnswer)
49454945
}
4946+
4947+
test("SPARK-51614: Having operator is properly resolved when there's generator in condition") {
4948+
val df = sql(
4949+
"""select
4950+
| explode(packages) as package
4951+
|from
4952+
| values(array('a')) t(packages)
4953+
|group by all
4954+
|having package in ('a')""".stripMargin
4955+
)
4956+
4957+
checkAnswer(df, Row("a"))
4958+
}
49464959
}
49474960

49484961
case class Foo(bar: Option[String])

0 commit comments

Comments
 (0)