Skip to content

Commit 0a47aae

Browse files
committed
Fix case insensitive resolution of GetField.
1 parent 3d0c37b commit 0a47aae

File tree

3 files changed

+28
-1
lines changed

3 files changed

+28
-1
lines changed

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

Lines changed: 10 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -21,6 +21,7 @@ import org.apache.spark.sql.catalyst.errors.TreeNodeException
2121
import org.apache.spark.sql.catalyst.expressions._
2222
import org.apache.spark.sql.catalyst.plans.logical._
2323
import org.apache.spark.sql.catalyst.rules._
24+
import org.apache.spark.sql.catalyst.types.StructType
2425

2526
/**
2627
* A trivial [[Analyzer]] with an [[EmptyCatalog]] and [[EmptyFunctionRegistry]]. Used for testing
@@ -187,6 +188,15 @@ class Analyzer(catalog: Catalog,
187188
val result = q.resolveChildren(name, resolver).getOrElse(u)
188189
logDebug(s"Resolving $u to $result")
189190
result
191+
192+
// Resolve field names using the resolver.
193+
case f @ GetField(child, fieldName) if !f.resolved && child.resolved =>
194+
child.dataType match {
195+
case StructType(fields) =>
196+
val resolvedFieldName = fields.map(_.name).find(resolver(_, fieldName))
197+
resolvedFieldName.map(n => f.copy(fieldName = n)).getOrElse(f)
198+
case _ => f
199+
}
190200
}
191201
}
192202

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/complexTypes.scala

Lines changed: 7 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -92,7 +92,13 @@ case class GetField(child: Expression, fieldName: String) extends UnaryExpressio
9292

9393
lazy val ordinal = structType.fields.indexOf(field)
9494

95-
override lazy val resolved = childrenResolved && child.dataType.isInstanceOf[StructType]
95+
override lazy val resolved = childrenResolved && fieldResolved
96+
97+
/** Returns true only if the fieldName is found in the child struct. */
98+
private def fieldResolved = child.dataType match {
99+
case StructType(fields) => fields.map(_.name).contains(fieldName)
100+
case _ => false
101+
}
96102

97103
override def eval(input: Row): Any = {
98104
val baseValue = child.eval(input).asInstanceOf[Row]

sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveResolutionSuite.scala

Lines changed: 11 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -27,6 +27,17 @@ case class Data(a: Int, B: Int, n: Nested, nestedArray: Seq[Nested])
2727
* A set of test cases expressed in Hive QL that are not covered by the tests included in the hive distribution.
2828
*/
2929
class HiveResolutionSuite extends HiveComparisonTest {
30+
31+
case class NestedData(a: Seq[NestedData2], B: NestedData2)
32+
case class NestedData2(a: NestedData3, B: NestedData3)
33+
case class NestedData3(a: Int, B: Int)
34+
35+
test("SPARK-3698: case insensitive test for nested data") {
36+
sparkContext.makeRDD(Seq.empty[NestedData]).registerTempTable("nested")
37+
// This should be successfully analyzed
38+
sql("SELECT a[0].A.A from nested").queryExecution.analyzed
39+
}
40+
3041
createQueryTest("table.attr",
3142
"SELECT src.key FROM src ORDER BY key LIMIT 1")
3243

0 commit comments

Comments
 (0)