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 @@ -136,7 +136,7 @@ package object dsl {
implicit def longToLiteral(l: Long): Literal = Literal(l)
implicit def floatToLiteral(f: Float): Literal = Literal(f)
implicit def doubleToLiteral(d: Double): Literal = Literal(d)
implicit def stringToLiteral(s: String): Literal = Literal(s)
implicit def stringToLiteral(s: String): Literal = Literal.create(s, StringType)
implicit def dateToLiteral(d: Date): Literal = Literal(d)
implicit def bigDecimalToLiteral(d: BigDecimal): Literal = Literal(d.underlying())
implicit def bigDecimalToLiteral(d: java.math.BigDecimal): Literal = Literal(d)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -28,7 +28,7 @@ import org.apache.spark.{SparkConf, SparkFunSuite}
import org.apache.spark.serializer.JavaSerializer
import org.apache.spark.sql.Row
import org.apache.spark.sql.catalyst.{CatalystTypeConverters, InternalRow}
import org.apache.spark.sql.catalyst.analysis.{ResolveTimeZone, SimpleAnalyzer}
import org.apache.spark.sql.catalyst.analysis.ResolveTimeZone
import org.apache.spark.sql.catalyst.expressions.codegen._
import org.apache.spark.sql.catalyst.optimizer.SimpleTestOptimizer
import org.apache.spark.sql.catalyst.plans.PlanTestBase
Expand Down Expand Up @@ -70,7 +70,9 @@ trait ExpressionEvalHelper extends GeneratorDrivenPropertyChecks with PlanTestBa
private def prepareEvaluation(expression: Expression): Expression = {
val serializer = new JavaSerializer(new SparkConf()).newInstance
val resolver = ResolveTimeZone(new SQLConf)
resolver.resolveTimeZones(serializer.deserialize(serializer.serialize(expression)))
val expr = resolver.resolveTimeZones(expression)
assert(expr.resolved)
serializer.deserialize(serializer.serialize(expr))
}

protected def checkEvaluation(
Expand Down Expand Up @@ -296,9 +298,7 @@ trait ExpressionEvalHelper extends GeneratorDrivenPropertyChecks with PlanTestBa
expected: Any,
inputRow: InternalRow = EmptyRow): Unit = {
val plan = Project(Alias(expression, s"Optimized($expression)")() :: Nil, OneRowRelation())
// We should analyze the plan first, otherwise we possibly optimize an unresolved plan.
val analyzedPlan = SimpleAnalyzer.execute(plan)
val optimizedPlan = SimpleTestOptimizer.execute(analyzedPlan)
val optimizedPlan = SimpleTestOptimizer.execute(plan)
checkEvaluationWithoutCodegen(optimizedPlan.expressions.head, expected, inputRow)
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -25,7 +25,7 @@ import org.scalatest.exceptions.TestFailedException
import org.apache.spark.{SparkException, SparkFunSuite}
import org.apache.spark.sql.AnalysisException
import org.apache.spark.sql.catalyst.InternalRow
import org.apache.spark.sql.catalyst.errors.TreeNodeException
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult
import org.apache.spark.sql.catalyst.plans.PlanTestBase
import org.apache.spark.sql.catalyst.util._
import org.apache.spark.sql.internal.SQLConf
Expand Down Expand Up @@ -694,11 +694,10 @@ class JsonExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper with
val mapType2 = MapType(IntegerType, CalendarIntervalType)
val schema2 = StructType(StructField("a", mapType2) :: Nil)
val struct2 = Literal.create(null, schema2)
intercept[TreeNodeException[_]] {
checkEvaluation(
StructsToJson(Map.empty, struct2, gmtId),
null
)
StructsToJson(Map.empty, struct2, gmtId).checkInputDataTypes() match {
case TypeCheckResult.TypeCheckFailure(msg) =>
assert(msg.contains("Unable to convert column a of type calendarinterval to JSON"))
case _ => fail("from_json should not work on interval map value type.")
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -24,6 +24,7 @@ import scala.collection.immutable.HashSet
import org.apache.spark.SparkFunSuite
import org.apache.spark.sql.RandomDataGenerator
import org.apache.spark.sql.catalyst.InternalRow
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult
import org.apache.spark.sql.catalyst.encoders.ExamplePointUDT
import org.apache.spark.sql.catalyst.expressions.codegen.CodegenContext
import org.apache.spark.sql.catalyst.util.{ArrayData, GenericArrayData}
Expand Down Expand Up @@ -231,22 +232,12 @@ class PredicateSuite extends SparkFunSuite with ExpressionEvalHelper {
testWithRandomDataGeneration(structType, nullable)
}

// Map types: not supported
for (
keyType <- atomicTypes;
valueType <- atomicTypes;
nullable <- Seq(true, false)) {
val mapType = MapType(keyType, valueType)
val e = intercept[Exception] {
testWithRandomDataGeneration(mapType, nullable)
}
if (e.getMessage.contains("Code generation of")) {
// If the `value` expression is null, `eval` will be short-circuited.
// Codegen version evaluation will be run then.
assert(e.getMessage.contains("cannot generate equality code for un-comparable type"))
} else {
assert(e.getMessage.contains("Exception evaluating"))
}
// In doesn't support map type and will fail the analyzer.
val map = Literal.create(create_map(1 -> 1), MapType(IntegerType, IntegerType))
In(map, Seq(map)).checkInputDataTypes() match {
case TypeCheckResult.TypeCheckFailure(msg) =>
assert(msg.contains("function in does not support ordering on type map"))
case _ => fail("In should not work on map type")
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -744,16 +744,14 @@ class StringExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper {

test("ParseUrl") {
def checkParseUrl(expected: String, urlStr: String, partToExtract: String): Unit = {
checkEvaluation(
ParseUrl(Seq(Literal(urlStr), Literal(partToExtract))), expected)
checkEvaluation(ParseUrl(Seq(urlStr, partToExtract)), expected)
}
def checkParseUrlWithKey(
expected: String,
urlStr: String,
partToExtract: String,
key: String): Unit = {
checkEvaluation(
ParseUrl(Seq(Literal(urlStr), Literal(partToExtract), Literal(key))), expected)
checkEvaluation(ParseUrl(Seq(urlStr, partToExtract, key)), expected)
}

checkParseUrl("spark.apache.org", "http://spark.apache.org/path?query=1", "HOST")
Expand Down Expand Up @@ -798,7 +796,6 @@ class StringExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper {
checkEvaluation(Sentences(nullString, nullString, nullString), null)
checkEvaluation(Sentences(nullString, nullString), null)
checkEvaluation(Sentences(nullString), null)
checkEvaluation(Sentences(Literal.create(null, NullType)), null)
checkEvaluation(Sentences("", nullString, nullString), Seq.empty)
checkEvaluation(Sentences("", nullString), Seq.empty)
checkEvaluation(Sentences(""), Seq.empty)
Expand Down