Skip to content

Commit

Permalink
throw exception when complex data type
Browse files Browse the repository at this point in the history
  • Loading branch information
AngersZhuuuu committed Jul 22, 2020
1 parent 4615733 commit 08d97c8
Show file tree
Hide file tree
Showing 2 changed files with 43 additions and 1 deletion.
Original file line number Diff line number Diff line change
Expand Up @@ -216,6 +216,9 @@ trait BaseScriptTransformationExec extends UnaryExecNode {
converter)
case udt: UserDefinedType[_] =>
wrapperConvertException(data => udt.deserialize(data), converter)
case ArrayType(_, _) | MapType(_, _, _) | StructType(_) =>
throw new SparkException("TRANSFORM without serde don't support" +
" ArrayType/MapType/StructType as output data type")
case _ => wrapperConvertException(data => data, converter)
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,7 +17,7 @@

package org.apache.spark.sql.execution

import org.apache.spark.TestUtils
import org.apache.spark.{SparkException, TestUtils}
import org.apache.spark.sql.catalyst.expressions.{Attribute, Expression}
import org.apache.spark.sql.catalyst.parser.ParseException
import org.apache.spark.sql.test.SharedSparkSession
Expand Down Expand Up @@ -61,4 +61,43 @@ class SparkScriptTransformationSuite extends BaseScriptTransformationSuite with
assert(e.contains("TRANSFORM with serde is only supported in hive mode"))
}
}

test("TRANSFORM don't support ArrayType/MapType/StructType as output data type (no serde)") {
assume(TestUtils.testCommandAvailable("/bin/bash"))
// check for ArrayType
val e1 = intercept[SparkException] {
sql(
"""
|SELECT TRANSFORM(a)
|USING 'cat' AS (a array<int>)
|FROM VALUES (array(1, 1), map('1', 1), struct(1, 'a')) t(a, b, c)
""".stripMargin).collect()
}.getMessage
assert(e1.contains("TRANSFORM without serde don't support" +
" ArrayType/MapType/StructType as output data type"))

// check for MapType
val e2 = intercept[SparkException] {
sql(
"""
|SELECT TRANSFORM(b)
|USING 'cat' AS (b map<int, string>)
|FROM VALUES (array(1, 1), map('1', 1), struct(1, 'a')) t(a, b, c)
""".stripMargin).collect()
}.getMessage
assert(e2.contains("TRANSFORM without serde don't support" +
" ArrayType/MapType/StructType as output data type"))

// check for StructType
val e3 = intercept[SparkException] {
sql(
"""
|SELECT TRANSFORM(c)
|USING 'cat' AS (c struct<col1:int, col2:string>)
|FROM VALUES (array(1, 1), map('1', 1), struct(1, 'a')) t(a, b, c)
""".stripMargin).collect()
}.getMessage
assert(e3.contains("TRANSFORM without serde don't support" +
" ArrayType/MapType/StructType as output data type"))
}
}

0 comments on commit 08d97c8

Please sign in to comment.