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
3 changes: 3 additions & 0 deletions python/pyspark/sql/types.py
Original file line number Diff line number Diff line change
Expand Up @@ -110,6 +110,9 @@ class NullType(DataType):

__metaclass__ = DataTypeSingleton

def simpleString(self):
return 'void'


class AtomicType(DataType):
"""An internal type used to represent everything that is not
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -32,6 +32,8 @@ class NullType private() extends DataType {
override def defaultSize: Int = 1

private[spark] override def asNullable: NullType = this

override def simpleString: String = "void"
}

/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -34,7 +34,7 @@
| org.apache.spark.sql.catalyst.expressions.Ascii | ascii | SELECT ascii('222') | struct<ascii(222):int> |
| org.apache.spark.sql.catalyst.expressions.Asin | asin | SELECT asin(0) | struct<ASIN(CAST(0 AS DOUBLE)):double> |
| org.apache.spark.sql.catalyst.expressions.Asinh | asinh | SELECT asinh(0) | struct<ASINH(CAST(0 AS DOUBLE)):double> |
| org.apache.spark.sql.catalyst.expressions.AssertTrue | assert_true | SELECT assert_true(0 < 1) | struct<assert_true((0 < 1)):null> |
| org.apache.spark.sql.catalyst.expressions.AssertTrue | assert_true | SELECT assert_true(0 < 1) | struct<assert_true((0 < 1)):void> |
| org.apache.spark.sql.catalyst.expressions.Atan | atan | SELECT atan(0) | struct<ATAN(CAST(0 AS DOUBLE)):double> |
| org.apache.spark.sql.catalyst.expressions.Atan2 | atan2 | SELECT atan2(0, 0) | struct<ATAN2(CAST(0 AS DOUBLE), CAST(0 AS DOUBLE)):double> |
| org.apache.spark.sql.catalyst.expressions.Atanh | atanh | SELECT atanh(0) | struct<ATANH(CAST(0 AS DOUBLE)):double> |
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -5,7 +5,7 @@
-- !query
select null, Null, nUll
-- !query schema
struct<NULL:null,NULL:null,NULL:null>
struct<NULL:void,NULL:void,NULL:void>
-- !query output
NULL NULL NULL

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -49,7 +49,7 @@ two 2
-- !query
select * from values ("one", null), ("two", null) as data(a, b)
-- !query schema
struct<a:string,b:null>
struct<a:string,b:void>
-- !query output
one NULL
two NULL
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -5,7 +5,7 @@
-- !query
select null, Null, nUll
-- !query schema
struct<NULL:null,NULL:null,NULL:null>
struct<NULL:void,NULL:void,NULL:void>
-- !query output
NULL NULL NULL

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -7,7 +7,7 @@ select typeof(null)
-- !query schema
struct<typeof(NULL):string>
-- !query output
null
void


-- !query
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -308,15 +308,15 @@ struct<1:int>
-- !query
select foo.* from (select null) as foo
-- !query schema
struct<NULL:null>
struct<NULL:void>
-- !query output
NULL


-- !query
select foo.* from (select 'xyzzy',1,null) as foo
-- !query schema
struct<xyzzy:string,1:int,NULL:null>
struct<xyzzy:string,1:int,NULL:void>
-- !query output
xyzzy 1 NULL

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -5,7 +5,7 @@
-- !query
SELECT ifnull(null, 'x'), ifnull('y', 'x'), ifnull(null, null)
-- !query schema
struct<ifnull(NULL, x):string,ifnull(y, x):string,ifnull(NULL, NULL):null>
struct<ifnull(NULL, x):string,ifnull(y, x):string,ifnull(NULL, NULL):void>
-- !query output
x y NULL

Expand All @@ -21,15 +21,15 @@ NULL x
-- !query
SELECT nvl(null, 'x'), nvl('y', 'x'), nvl(null, null)
-- !query schema
struct<nvl(NULL, x):string,nvl(y, x):string,nvl(NULL, NULL):null>
struct<nvl(NULL, x):string,nvl(y, x):string,nvl(NULL, NULL):void>
-- !query output
x y NULL


-- !query
SELECT nvl2(null, 'x', 'y'), nvl2('n', 'x', 'y'), nvl2(null, null, null)
-- !query schema
struct<nvl2(NULL, x, y):string,nvl2(n, x, y):string,nvl2(NULL, NULL, NULL):null>
struct<nvl2(NULL, x, y):string,nvl2(n, x, y):string,nvl2(NULL, NULL, NULL):void>
-- !query output
y x NULL

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -49,7 +49,7 @@ two 2
-- !query
select udf(a), b from values ("one", null), ("two", null) as data(a, b)
-- !query schema
struct<CAST(udf(cast(a as string)) AS STRING):string,b:null>
struct<CAST(udf(cast(a as string)) AS STRING):string,b:void>
-- !query output
one NULL
two NULL
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -405,7 +405,7 @@ class FileBasedDataSourceSuite extends QueryTest
""
}
def errorMessage(format: String): String = {
s"$format data source does not support null data type."
s"$format data source does not support ${NullType.simpleString} data type."
}
withSQLConf(SQLConf.USE_V1_SOURCE_LIST.key -> useV1List) {
withTempDir { dir =>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -2331,17 +2331,17 @@ class HiveDDLSuite
val e1 = intercept[AnalysisException] {
spark.sql("CREATE TABLE t1 USING PARQUET AS SELECT null as null_col")
}.getMessage
assert(e1.contains("Cannot create tables with null type"))
assert(e1.contains(s"Cannot create tables with ${NullType.simpleString} type"))

val e2 = intercept[AnalysisException] {
spark.sql("CREATE TABLE t2 AS SELECT null as null_col")
}.getMessage
assert(e2.contains("Cannot create tables with null type"))
assert(e2.contains(s"Cannot create tables with ${NullType.simpleString} type"))

val e3 = intercept[AnalysisException] {
spark.sql("CREATE TABLE t3 STORED AS PARQUET AS SELECT null as null_col")
}.getMessage
assert(e3.contains("Cannot create tables with null type"))
assert(e3.contains(s"Cannot create tables with ${NullType.simpleString} type"))
}

// Forbid Replace table AS SELECT with null type
Expand All @@ -2350,27 +2350,27 @@ class HiveDDLSuite
val e = intercept[AnalysisException] {
spark.sql(s"CREATE OR REPLACE TABLE t USING $v2Source AS SELECT null as null_col")
}.getMessage
assert(e.contains("Cannot create tables with null type"))
assert(e.contains(s"Cannot create tables with ${NullType.simpleString} type"))
}

// Forbid creating table with VOID type in Spark
withTable("t1", "t2", "t3", "t4") {
val e1 = intercept[AnalysisException] {
spark.sql(s"CREATE TABLE t1 (v VOID) USING PARQUET")
}.getMessage
assert(e1.contains("Cannot create tables with null type"))
assert(e1.contains(s"Cannot create tables with ${NullType.simpleString} type"))
val e2 = intercept[AnalysisException] {
spark.sql(s"CREATE TABLE t2 (v VOID) USING hive")
}.getMessage
assert(e2.contains("Cannot create tables with null type"))
assert(e2.contains(s"Cannot create tables with ${NullType.simpleString} type"))
val e3 = intercept[AnalysisException] {
spark.sql(s"CREATE TABLE t3 (v VOID)")
}.getMessage
assert(e3.contains("Cannot create tables with null type"))
assert(e3.contains(s"Cannot create tables with ${NullType.simpleString} type"))
val e4 = intercept[AnalysisException] {
spark.sql(s"CREATE TABLE t4 (v VOID) STORED AS PARQUET")
}.getMessage
assert(e4.contains("Cannot create tables with null type"))
assert(e4.contains(s"Cannot create tables with ${NullType.simpleString} type"))
}

// Forbid Replace table with VOID type
Expand All @@ -2379,7 +2379,7 @@ class HiveDDLSuite
val e = intercept[AnalysisException] {
spark.sql(s"CREATE OR REPLACE TABLE t (v VOID) USING $v2Source")
}.getMessage
assert(e.contains("Cannot create tables with null type"))
assert(e.contains(s"Cannot create tables with ${NullType.simpleString} type"))
}

// Make sure spark.catalog.createTable with null type will fail
Expand Down Expand Up @@ -2416,7 +2416,7 @@ class HiveDDLSuite
schema = schema,
options = Map("fileFormat" -> "parquet"))
}.getMessage
assert(e.contains("Cannot create tables with null type"))
assert(e.contains(s"Cannot create tables with ${NullType.simpleString} type"))
}
}

Expand All @@ -2429,7 +2429,7 @@ class HiveDDLSuite
schema = schema,
options = Map.empty[String, String])
}.getMessage
assert(e.contains("Cannot create tables with null type"))
assert(e.contains(s"Cannot create tables with ${NullType.simpleString} type"))
}
}

Expand Down Expand Up @@ -2854,4 +2854,24 @@ class HiveDDLSuite
assert(sql("SELECT * FROM t2 WHERE c = 'A'").collect().isEmpty)
}
}

test("SPARK-32445: Make NullType.simpleString as void to support hive") {
withView("v1") {
sql("create view v1 as select null as c")
val df = sql("select * from v1")
assert(df.schema.fields.head.dataType == NullType)
checkAnswer(
df,
Row(null)
)

sql("alter view v1 as select null as c1, 1 as c2")
val df2 = sql("select * from v1")
assert(df2.schema.fields.head.dataType == NullType)
checkAnswer(
df2,
Row(null, 1)
)
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -121,7 +121,7 @@ class HiveOrcSourceSuite extends OrcSuite with TestHiveSingleton {
msg = intercept[AnalysisException] {
sql("select null").write.mode("overwrite").orc(orcDir)
}.getMessage
assert(msg.contains("ORC data source does not support null data type."))
assert(msg.contains(s"ORC data source does not support ${NullType.simpleString} data type."))

msg = intercept[AnalysisException] {
spark.udf.register("testType", () => new IntervalData())
Expand Down