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
6 changes: 4 additions & 2 deletions docs/sql-migration-guide.md
Original file line number Diff line number Diff line change
Expand Up @@ -9,9 +9,9 @@ license: |
The ASF licenses this file to You under the Apache License, Version 2.0
(the "License"); you may not use this file except in compliance with
the License. You may obtain a copy of the License at

http://www.apache.org/licenses/LICENSE-2.0

Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
Expand Down Expand Up @@ -218,6 +218,8 @@ license: |

- Since Spark 3.0, the `size` function returns `NULL` for the `NULL` input. In Spark version 2.4 and earlier, this function gives `-1` for the same input. To restore the behavior before Spark 3.0, you can set `spark.sql.legacy.sizeOfNull` to `true`.

- Since Spark 3.0, the interval literal syntax does not allow multiple from-to units anymore. For example, `SELECT INTERVAL '1-1' YEAR TO MONTH '2-2' YEAR TO MONTH'` throws parser exception.

## Upgrading from Spark SQL 2.4 to 2.4.1

- The value of `spark.executor.heartbeatInterval`, when specified without units like "30" rather than "30s", was
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -80,7 +80,7 @@ singleTableSchema
;

singleInterval
: INTERVAL? (intervalValue intervalUnit)+ EOF
: INTERVAL? multiUnitsInterval EOF
;

statement
Expand Down Expand Up @@ -759,12 +759,24 @@ booleanValue
;

interval
: {ansi}? INTERVAL? intervalField+
| {!ansi}? INTERVAL intervalField*
: INTERVAL (errorCapturingMultiUnitsInterval | errorCapturingUnitToUnitInterval)?
| {ansi}? (errorCapturingMultiUnitsInterval | errorCapturingUnitToUnitInterval)
Copy link
Contributor Author

Choose a reason for hiding this comment

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

{ansi}? (errorCapturingMultiUnitsInterval | errorCapturingUnitToUnitInterval)? is illegal as it can match anything. Note that we need to make (errorCapturingMultiUnitsInterval | errorCapturingUnitToUnitInterval) optional so that we can detect select interval and give precise error message.

;

intervalField
: value=intervalValue unit=intervalUnit (TO to=intervalUnit)?
errorCapturingMultiUnitsInterval
: multiUnitsInterval unitToUnitInterval?
;

multiUnitsInterval
: (intervalValue intervalUnit)+
;

errorCapturingUnitToUnitInterval
: body=unitToUnitInterval (error1=multiUnitsInterval | error2=unitToUnitInterval)?
;

unitToUnitInterval
: value=intervalValue from=intervalUnit TO to=intervalUnit
;

intervalValue
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -102,20 +102,7 @@ class AstBuilder(conf: SQLConf) extends SqlBaseBaseVisitor[AnyRef] with Logging
}

override def visitSingleInterval(ctx: SingleIntervalContext): CalendarInterval = {
withOrigin(ctx) {
val units = ctx.intervalUnit().asScala.map {
u => normalizeInternalUnit(u.getText.toLowerCase(Locale.ROOT))
}.toArray
val values = ctx.intervalValue().asScala.map(getIntervalValue).toArray
try {
IntervalUtils.fromUnitStrings(units, values)
} catch {
case i: IllegalArgumentException =>
val e = new ParseException(i.getMessage, ctx)
e.setStackTrace(i.getStackTrace)
throw e
}
}
withOrigin(ctx)(visitMultiUnitsInterval(ctx.multiUnitsInterval))
}

/* ********************************************************************************************
Expand Down Expand Up @@ -1940,71 +1927,102 @@ class AstBuilder(conf: SQLConf) extends SqlBaseBaseVisitor[AnyRef] with Logging
}

/**
* Create a [[CalendarInterval]] literal expression. An interval expression can contain multiple
* unit value pairs, for instance: interval 2 months 2 days.
* Create a [[CalendarInterval]] literal expression. Two syntaxes are supported:
* - multiple unit value pairs, for instance: interval 2 months 2 days.
* - from-to unit, for instance: interval '1-2' year to month.
*/
override def visitInterval(ctx: IntervalContext): Literal = withOrigin(ctx) {
val intervals = ctx.intervalField.asScala.map(visitIntervalField)
validate(intervals.nonEmpty, "at least one time unit should be given for interval literal", ctx)
Literal(intervals.reduce(_.add(_)))
if (ctx.errorCapturingMultiUnitsInterval != null) {
val innerCtx = ctx.errorCapturingMultiUnitsInterval
if (innerCtx.unitToUnitInterval != null) {
throw new ParseException(
"Can only have a single from-to unit in the interval literal syntax",
innerCtx.unitToUnitInterval)
}
Literal(visitMultiUnitsInterval(innerCtx.multiUnitsInterval), CalendarIntervalType)
} else if (ctx.errorCapturingUnitToUnitInterval != null) {
val innerCtx = ctx.errorCapturingUnitToUnitInterval
if (innerCtx.error1 != null || innerCtx.error2 != null) {
val errorCtx = if (innerCtx.error1 != null) innerCtx.error1 else innerCtx.error2
throw new ParseException(
"Can only have a single from-to unit in the interval literal syntax",
errorCtx)
}
Literal(visitUnitToUnitInterval(innerCtx.body), CalendarIntervalType)
} else {
throw new ParseException("at least one time unit should be given for interval literal", ctx)
}
}

/**
* Create a [[CalendarInterval]] for a unit value pair. Two unit configuration types are
* supported:
* - Single unit.
* - From-To unit ('YEAR TO MONTH', 'DAY TO HOUR', 'DAY TO MINUTE', 'DAY TO SECOND',
* 'HOUR TO MINUTE', 'HOUR TO SECOND' and 'MINUTE TO SECOND' are supported).
* Creates a [[CalendarInterval]] with multiple unit value pairs, e.g. 1 YEAR 2 DAYS.
*/
override def visitIntervalField(ctx: IntervalFieldContext): CalendarInterval = withOrigin(ctx) {
import ctx._
val s = getIntervalValue(value)
try {
val unitText = unit.getText.toLowerCase(Locale.ROOT)
val interval = (unitText, Option(to).map(_.getText.toLowerCase(Locale.ROOT))) match {
case (u, None) =>
IntervalUtils.fromUnitStrings(Array(normalizeInternalUnit(u)), Array(s))
case ("year", Some("month")) =>
IntervalUtils.fromYearMonthString(s)
case ("day", Some("hour")) =>
IntervalUtils.fromDayTimeString(s, "day", "hour")
case ("day", Some("minute")) =>
IntervalUtils.fromDayTimeString(s, "day", "minute")
case ("day", Some("second")) =>
IntervalUtils.fromDayTimeString(s, "day", "second")
case ("hour", Some("minute")) =>
IntervalUtils.fromDayTimeString(s, "hour", "minute")
case ("hour", Some("second")) =>
IntervalUtils.fromDayTimeString(s, "hour", "second")
case ("minute", Some("second")) =>
IntervalUtils.fromDayTimeString(s, "minute", "second")
case (from, Some(t)) =>
throw new ParseException(s"Intervals FROM $from TO $t are not supported.", ctx)
override def visitMultiUnitsInterval(ctx: MultiUnitsIntervalContext): CalendarInterval = {
withOrigin(ctx) {
val units = ctx.intervalUnit().asScala.map { unit =>
val u = unit.getText.toLowerCase(Locale.ROOT)
// Handle plural forms, e.g: yearS/monthS/weekS/dayS/hourS/minuteS/hourS/...
if (u.endsWith("s")) u.substring(0, u.length - 1) else u
}.toArray

val values = ctx.intervalValue().asScala.map { value =>
if (value.STRING() != null) {
string(value.STRING())
} else {
value.getText
}
}.toArray

try {
IntervalUtils.fromUnitStrings(units, values)
} catch {
case i: IllegalArgumentException =>
val e = new ParseException(i.getMessage, ctx)
e.setStackTrace(i.getStackTrace)
throw e
}
validate(interval != null, "No interval can be constructed", ctx)
interval
} catch {
// Handle Exceptions thrown by CalendarInterval
case e: IllegalArgumentException =>
val pe = new ParseException(e.getMessage, ctx)
pe.setStackTrace(e.getStackTrace)
throw pe
}
}

private def getIntervalValue(value: IntervalValueContext): String = {
if (value.STRING() != null) {
string(value.STRING())
} else {
value.getText
/**
* Creates a [[CalendarInterval]] with from-to unit, e.g. '2-1' YEAR TO MONTH.
*/
override def visitUnitToUnitInterval(ctx: UnitToUnitIntervalContext): CalendarInterval = {
withOrigin(ctx) {
val value = Option(ctx.intervalValue.STRING).map(string).getOrElse {
throw new ParseException("The value of from-to unit must be a string", ctx.intervalValue)
}
try {
val from = ctx.from.getText.toLowerCase(Locale.ROOT)
val to = ctx.to.getText.toLowerCase(Locale.ROOT)
(from, to) match {
case ("year", "month") =>
IntervalUtils.fromYearMonthString(value)
case ("day", "hour") =>
IntervalUtils.fromDayTimeString(value, "day", "hour")
case ("day", "minute") =>
IntervalUtils.fromDayTimeString(value, "day", "minute")
case ("day", "second") =>
IntervalUtils.fromDayTimeString(value, "day", "second")
case ("hour", "minute") =>
IntervalUtils.fromDayTimeString(value, "hour", "minute")
case ("hour", "second") =>
IntervalUtils.fromDayTimeString(value, "hour", "second")
case ("minute", "second") =>
IntervalUtils.fromDayTimeString(value, "minute", "second")
case _ =>
throw new ParseException(s"Intervals FROM $from TO $to are not supported.", ctx)
}
} catch {
// Handle Exceptions thrown by CalendarInterval
case e: IllegalArgumentException =>
val pe = new ParseException(e.getMessage, ctx)
pe.setStackTrace(e.getStackTrace)
throw pe
}
}
}

// Handle plural forms, e.g: yearS/monthS/weekS/dayS/hourS/minuteS/hourS/...
private def normalizeInternalUnit(s: String): String = {
if (s.endsWith("s")) s.substring(0, s.length - 1) else s
}

/* ********************************************************************************************
* DataType parsing
* ******************************************************************************************** */
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -644,7 +644,7 @@ class ExpressionParserSuite extends AnalysisTest {

// Non Existing unit
intercept("interval 10 nanoseconds",
"no viable alternative at input 'interval 10 nanoseconds'")
"no viable alternative at input '10 nanoseconds'")

// Year-Month intervals.
val yearMonthValues = Seq("123-10", "496-0", "-2-3", "-123-0")
Expand Down Expand Up @@ -679,16 +679,13 @@ class ExpressionParserSuite extends AnalysisTest {
}

// Unknown FROM TO intervals
intercept("interval 10 month to second",
intercept("interval '10' month to second",
Copy link
Contributor Author

Choose a reason for hiding this comment

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

from-to unit can only handle string values ('1-2' year to month, '1 12' day to hour).

Copy link
Contributor Author

@cloud-fan cloud-fan Oct 30, 2019

Choose a reason for hiding this comment

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

If you do write interval 10 month to second, then we will ask users to use string instead, see https://github.com/apache/spark/pull/26285/files#diff-4f9e28af8e9fcb40a8a99b4e49f3b9b2R612

"Intervals FROM month TO second are not supported.")

// Composed intervals.
checkIntervals(
"3 months 4 days 22 seconds 1 millisecond",
Literal(new CalendarInterval(3, 4, 22001000L)))
checkIntervals(
"3 years '-1-10' year to month 3 weeks '1 0:0:2' day to second",
Literal(new CalendarInterval(14, 22, 2 * CalendarInterval.MICROS_PER_SECOND)))
}

test("SPARK-23264 Interval Compatibility tests") {
Expand Down
23 changes: 23 additions & 0 deletions sql/core/src/test/resources/sql-tests/inputs/literals.sql
Original file line number Diff line number Diff line change
Expand Up @@ -130,3 +130,26 @@ select interval '3 year 1 hour';
select integer '7';
select integer'7';
select integer '2147483648';

-- malformed interval literal
select interval;
select interval 1 fake_unit;
select interval 1 year to month;
select interval '1' year to second;
select interval '10-9' year to month '2-1' year to month;
select interval '10-9' year to month '12:11:10' hour to second;
select interval '1 15:11' day to minute '12:11:10' hour to second;
select interval 1 year '2-1' year to month;
select interval 1 year '12:11:10' hour to second;
select interval '10-9' year to month '1' year;
select interval '12:11:10' hour to second '1' year;
-- malformed interval literal with ansi mode
SET spark.sql.ansi.enabled=true;
select interval;
select interval 1 fake_unit;
select interval 1 year to month;
select 1 year to month;
select interval '1' year to second;
select '1' year to second;
select interval 1 year '2-1' year to month;
select 1 year '2-1' year to month;
Loading