Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -1161,13 +1161,21 @@ class Analyzer(
* Replaces [[UnresolvedFunction]]s with concrete [[Expression]]s.
*/
object ResolveFunctions extends Rule[LogicalPlan] {
private def isGroupingAnalyticsOp(plan: LogicalPlan): Boolean = plan match {
case Aggregate(Seq(c @ Cube(groupByExprs)), _, _) => true
case Aggregate(Seq(r @ Rollup(groupByExprs)), _, _) => true
case x: GroupingSets => true
case _ => false
}

def apply(plan: LogicalPlan): LogicalPlan = plan.resolveOperators {
case q: LogicalPlan =>
q transformExpressions {
case u if !u.childrenResolved => u // Skip until children are resolved.
case u: UnresolvedAttribute if resolver(u.name, VirtualColumn.hiveGroupingIdName) =>
case u: UnresolvedAttribute
if resolver(u.name, VirtualColumn.groupingIdName) && isGroupingAnalyticsOp(q) =>
withPosition(u) {
Alias(GroupingID(Nil), VirtualColumn.hiveGroupingIdName)()
Alias(GroupingID(Nil), VirtualColumn.groupingIdName)()
}
case u @ UnresolvedGenerator(name, children) =>
withPosition(u) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -367,8 +367,6 @@ case class OuterReference(e: NamedExpression)
}

object VirtualColumn {
// The attribute name used by Hive, which has different result than Spark, deprecated.
val hiveGroupingIdName: String = "grouping__id"
val groupingIdName: String = "spark_grouping_id"
val groupingIdName: String = "grouping__id"
val groupingIdAttribute: UnresolvedAttribute = UnresolvedAttribute(groupingIdName)
}
Original file line number Diff line number Diff line change
Expand Up @@ -32,8 +32,7 @@ class ResolveGroupingAnalyticsSuite extends AnalysisTest {
lazy val unresolved_a = UnresolvedAttribute("a")
lazy val unresolved_b = UnresolvedAttribute("b")
lazy val unresolved_c = UnresolvedAttribute("c")
lazy val gid = 'spark_grouping_id.int.withNullability(false)
lazy val hive_gid = 'grouping__id.int.withNullability(false)
lazy val gid = 'grouping__id.int.withNullability(false)
lazy val grouping_a = Cast(ShiftRight(gid, 1) & 1, ByteType, Option(TimeZone.getDefault().getID))
lazy val nulInt = Literal(null, IntegerType)
lazy val nulStr = Literal(null, StringType)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1497,6 +1497,27 @@ class SQLQuerySuite extends QueryTest with SQLTestUtils with TestHiveSingleton {
}
}

test("select grouping__id from subquery.") {
checkAnswer(
sql(
"""
|SELECT cnt, k2, k3, grouping__id
|FROM
| (SELECT count(*) as cnt, k2, k3, grouping__id
Copy link
Member

Choose a reason for hiding this comment

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

SELECT cnt, k2, k3, alias_grouping__id
FROM
  (SELECT count(*) as cnt, k2, k3, grouping__id as alias_grouping__id
  FROM (SELECT key, key%2 as k2 , key%3 as k3 FROM src) t1
  GROUP BY k2, k3
  GROUPING SETS(k2, k3)) t2
ORDER BY alias_grouping__id, k2, k3

This is the workaround.

| FROM (SELECT key, key%2 as k2 , key%3 as k3 FROM src) t1
| GROUP BY k2, k3
| GROUPING SETS(k2, k3)) t2
|ORDER BY grouping__id, k2, k3
""".stripMargin),
Seq(
(247, 0, null, 1),
(253, 1, null, 1),
(169, null, 0, 2),
(165, null, 1, 2),
(166, null, 2, 2)
).map(i => Row(i._1, i._2, i._3, i._4)))
}

ignore("SPARK-10562: partition by column with mixed case name") {
withTable("tbl10562") {
val df = Seq(2012 -> "a").toDF("Year", "val")
Expand Down