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 @@ -324,9 +324,11 @@ object TypeCoercion {
// Skip nodes who's children have not been resolved yet.
case e if !e.childrenResolved => e

case a @ BinaryArithmetic(left @ StringType(), right) =>
case a @ BinaryArithmetic(left @ StringType(), right)
if right.dataType != CalendarIntervalType =>
a.makeCopy(Array(Cast(left, DoubleType), right))
case a @ BinaryArithmetic(left, right @ StringType()) =>
case a @ BinaryArithmetic(left, right @ StringType())
if left.dataType != CalendarIntervalType =>
a.makeCopy(Array(left, Cast(right, DoubleType)))

// For equality between string and timestamp we cast the string to a timestamp
Expand Down
13 changes: 12 additions & 1 deletion sql/core/src/test/scala/org/apache/spark/sql/SQLQuerySuite.scala
Original file line number Diff line number Diff line change
Expand Up @@ -20,7 +20,7 @@ package org.apache.spark.sql
import java.io.File
import java.math.MathContext
import java.net.{MalformedURLException, URL}
import java.sql.Timestamp
import java.sql.{Date, Timestamp}
import java.util.concurrent.atomic.AtomicBoolean

import org.apache.spark.{AccumulatorSuite, SparkException}
Expand Down Expand Up @@ -2760,6 +2760,17 @@ class SQLQuerySuite extends QueryTest with SharedSQLContext {
}
}

test("SPARK-22894: DateTimeOperations should accept SQL like string type") {
Copy link
Member

@gatorsmile gatorsmile Dec 25, 2017

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Add it to #20061?

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yes, I'll and it.

val date = "2017-12-24"
val str = sql(s"SELECT CAST('$date' as STRING) + interval 2 months 2 seconds")
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

So far, we issue an exception. What is the behavior of Hive?

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Hive doesn't accept string type:

hive> SELECT cast('2017-12-24' as date) + interval 2 day;
2017-12-26 00:00:00
hive> SELECT cast('2017-12-24' as timestamp) + interval 2 day;
2017-12-26 00:00:00
hive> SELECT cast('2017-12-24' as string) + interval 2 day;
FAILED: SemanticException Line 0:-1 Wrong arguments '2': No matching method for class org.apache.hadoop.hive.ql.udf.generic.GenericUDFOPDTIPlus with (string, interval_day_time)
hive>

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

But Spark was originally supported:

ruleTest(dateTimeOperations, Add(str, interval), Cast(TimeAdd(str, interval), StringType))

Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I saw the original PR https://github.com/apache/spark/pull/7754/files#r35821191

Maybe the SQL API should support it since we do support it in DataFrame APIs.

val dt = sql(s"SELECT CAST('$date' as DATE) + interval 2 months 2 seconds")
val ts = sql(s"SELECT CAST('$date' as TIMESTAMP) + interval 2 months 2 seconds")

checkAnswer(str, Row("2018-02-24 00:00:02") :: Nil)
checkAnswer(dt, Row(Date.valueOf("2018-02-24")) :: Nil)
checkAnswer(ts, Row(Timestamp.valueOf("2018-02-24 00:00:02")) :: Nil)
}
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Please get rid of this test case and merge them to #20061


// Only New OrcFileFormat supports this
Seq(classOf[org.apache.spark.sql.execution.datasources.orc.OrcFileFormat].getCanonicalName,
"parquet").foreach { format =>
Expand Down