|
17 | 17 |
|
18 | 18 | package org.apache.spark.sql.catalyst.analysis
|
19 | 19 |
|
20 |
| -import java.util.TimeZone |
| 20 | +import java.util.{Locale, TimeZone} |
21 | 21 |
|
22 | 22 | import scala.reflect.ClassTag
|
23 | 23 |
|
24 | 24 | import org.scalatest.Matchers
|
25 | 25 |
|
26 | 26 | import org.apache.spark.api.python.PythonEvalType
|
27 | 27 | import org.apache.spark.sql.catalyst.TableIdentifier
|
| 28 | +import org.apache.spark.sql.catalyst.catalog.{CatalogStorageFormat, CatalogTable, CatalogTableType} |
28 | 29 | import org.apache.spark.sql.catalyst.dsl.expressions._
|
29 | 30 | import org.apache.spark.sql.catalyst.dsl.plans._
|
30 | 31 | import org.apache.spark.sql.catalyst.expressions._
|
31 | 32 | import org.apache.spark.sql.catalyst.plans.{Cross, Inner}
|
32 | 33 | import org.apache.spark.sql.catalyst.plans.logical._
|
33 | 34 | import org.apache.spark.sql.catalyst.plans.physical.{HashPartitioning, Partitioning,
|
34 | 35 | RangePartitioning, RoundRobinPartitioning}
|
| 36 | +import org.apache.spark.sql.catalyst.rules.RuleExecutor |
35 | 37 | import org.apache.spark.sql.catalyst.util._
|
36 | 38 | import org.apache.spark.sql.internal.SQLConf
|
37 | 39 | import org.apache.spark.sql.types._
|
@@ -604,4 +606,28 @@ class AnalysisSuite extends AnalysisTest with Matchers {
|
604 | 606 | checkAnalysis(input, expected)
|
605 | 607 | }
|
606 | 608 | }
|
| 609 | + |
| 610 | + test("SPARK-25691: AliasViewChild with different nullabilities") { |
| 611 | + object ViewAnalyzer extends RuleExecutor[LogicalPlan] { |
| 612 | + val batches = Batch("View", Once, AliasViewChild(conf), EliminateView) :: Nil |
| 613 | + } |
| 614 | + def intNotNullableAttr(name: String): Attribute = { |
| 615 | + AttributeReference(name, IntegerType, nullable = false)() |
| 616 | + } |
| 617 | + val relation = LocalRelation(intNotNullableAttr("a"), 'b.string) |
| 618 | + val view = View(CatalogTable( |
| 619 | + identifier = TableIdentifier("v1"), |
| 620 | + tableType = CatalogTableType.VIEW, |
| 621 | + storage = CatalogStorageFormat.empty, |
| 622 | + schema = StructType(Seq(StructField("a", IntegerType), StructField("b", StringType)))), |
| 623 | + output = Seq('a.int, 'b.string), |
| 624 | + child = relation) |
| 625 | + val tz = Option(conf.sessionLocalTimeZone) |
| 626 | + val expected = Project(Seq( |
| 627 | + Alias(Cast(intNotNullableAttr("a"), IntegerType, tz), "a")(), |
| 628 | + Alias(Cast('b.string, StringType, tz), "b")()), |
| 629 | + relation) |
| 630 | + val res = ViewAnalyzer.execute(view) |
| 631 | + comparePlans(res, expected) |
| 632 | + } |
607 | 633 | }
|
0 commit comments