Skip to content
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@ import scala.util.control.NonFatal

import org.apache.spark.sql.catalyst.parser.{CatalystSqlParser, ParseException}
import org.apache.spark.sql.catalyst.util.DateTimeConstants._
import org.apache.spark.sql.internal.SQLConf
import org.apache.spark.sql.types.Decimal
import org.apache.spark.unsafe.types.{CalendarInterval, UTF8String}

Expand Down Expand Up @@ -203,7 +204,7 @@ object IntervalUtils {

try {
val sign = if (m.group(1) != null && m.group(1) == "-") -1 else 1
val days = if (m.group(2) == null) {
var days = if (m.group(2) == null) {
0
} else {
toLongWithRange(DAY, m.group(3), 0, Integer.MAX_VALUE).toInt
Expand All @@ -224,16 +225,27 @@ object IntervalUtils {
}
// Hive allow nanosecond precision interval
var secondsFraction = parseNanos(m.group(9), seconds < 0)
to match {
case HOUR =>
val resetValuesUpToFrom = !SQLConf.get.usePostgreSQLDialect
(from, to) match {
case (DAY, SECOND) => // No-op
case (DAY, MINUTE) =>
seconds = 0
secondsFraction = 0
case (DAY, HOUR) =>
minutes = 0
seconds = 0
secondsFraction = 0
case MINUTE =>
case (HOUR, SECOND) =>
if (resetValuesUpToFrom) days = 0
case (HOUR, MINUTE) =>
if (resetValuesUpToFrom) days = 0
seconds = 0
secondsFraction = 0
case SECOND =>
// No-op
case (MINUTE, SECOND) =>
if (resetValuesUpToFrom) {
days = 0
hours = 0
}
case _ =>
throw new IllegalArgumentException(
s"Cannot support (interval '$input' $from to $to) expression")
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,12 +20,14 @@ package org.apache.spark.sql.catalyst.util
import java.util.concurrent.TimeUnit

import org.apache.spark.SparkFunSuite
import org.apache.spark.sql.catalyst.plans.SQLHelper
import org.apache.spark.sql.catalyst.util.DateTimeConstants._
import org.apache.spark.sql.catalyst.util.IntervalUtils._
import org.apache.spark.sql.catalyst.util.IntervalUtils.IntervalUnit._
import org.apache.spark.sql.internal.SQLConf
import org.apache.spark.unsafe.types.{CalendarInterval, UTF8String}

class IntervalUtilsSuite extends SparkFunSuite {
class IntervalUtilsSuite extends SparkFunSuite with SQLHelper {

private def checkFromString(input: String, expected: CalendarInterval): Unit = {
assert(fromString(input) === expected)
Expand Down Expand Up @@ -265,4 +267,30 @@ class IntervalUtilsSuite extends SparkFunSuite {
assert(e.getMessage.contains("divide by zero"))
}
}

test("from bounded day-time string") {
val input = "5 12:40:30.999999999"

def check(from: IntervalUnit, to: IntervalUnit, expected: String): Unit = {
withClue(s"from = $from, to = $to") {
assert(fromDayTimeString(input, from, to) === fromString(expected))
}
}

withSQLConf(SQLConf.DIALECT.key -> "Spark") {
Copy link
Contributor

Choose a reason for hiding this comment

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

can we have a test case to show the error behavior?

Copy link
Member Author

Choose a reason for hiding this comment

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

Just to be clear, this PR just truncates results according to specified bound. In particular, it takes into account the left bound. Could you explain, please, what do you mean by saying error behavior?

Copy link
Contributor

Choose a reason for hiding this comment

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

I think the conclusion is to fail for invalid bounds?

Copy link
Member Author

Choose a reason for hiding this comment

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

Do you mean if a string doesn't match to specified bounds like in literals.sql: https://github.com/apache/spark/pull/26358/files#diff-4f9e28af8e9fcb40a8a99b4e49f3b9b2R424 ?

Copy link
Contributor

Choose a reason for hiding this comment

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

yea, like what Oracle does.

Copy link
Member Author

Choose a reason for hiding this comment

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

Here is the PR with strict parsing #26473

check(HOUR, MINUTE, "12 hours 40 minutes")
check(HOUR, SECOND, "12 hours 40 minutes 30.999999 seconds")
check(MINUTE, SECOND, "40 minutes 30.999999 seconds")
check(DAY, HOUR, "5 days 12 hours")
check(DAY, MINUTE, "5 days 12 hours 40 minutes")
}

withSQLConf(SQLConf.DIALECT.key -> "PostgreSQL") {
check(HOUR, MINUTE, "5 days 12 hours 40 minutes")
check(HOUR, SECOND, "5 days 12 hours 40 minutes 30.999999 seconds")
check(MINUTE, SECOND, "5 days 12 hours 40 minutes 30.999999 seconds")
check(DAY, HOUR, "5 days 12 hours")
check(DAY, MINUTE, "5 days 12 hours 40 minutes")
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -419,9 +419,9 @@ struct<15 hours 40 minutes 32.998999 seconds:interval>
-- !query 46
select interval '20 40:32.99899999' minute to second
-- !query 46 schema
struct<20 days 40 minutes 32.998999 seconds:interval>
struct<40 minutes 32.998999 seconds:interval>
-- !query 46 output
20 days 40 minutes 32.998999 seconds
40 minutes 32.998999 seconds


-- !query 47
Expand Down