Skip to content

Commit 17b0438

Browse files
committed
[SPARK-31586][SQL] Replace expression TimeSub(l, r) with TimeAdd(l -r)
1 parent ebc8fa5 commit 17b0438

File tree

12 files changed

+45
-91
lines changed

12 files changed

+45
-91
lines changed

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/Analyzer.scala

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -278,7 +278,7 @@ class Analyzer(
278278
* 1. if both side are interval, stays the same;
279279
* 2. else if the left side is date and the right side is interval,
280280
* turns it to [[DateAddInterval(l, -r)]];
281-
* 3. else if the right side is an interval, turns it to [[TimeSub]];
281+
* 3. else if the right side is an interval, turns it to [[TimeAdd(l, -r)]];
282282
* 4. else if one side is timestamp, turns it to [[SubtractTimestamps]];
283283
* 5. else if the right side is date, turns it to [[DateDiff]]/[[SubtractDates]];
284284
* 6. else if the left side is date, turns it to [[DateSub]];
@@ -308,7 +308,7 @@ class Analyzer(
308308
case s @ Subtract(l, r) if s.childrenResolved => (l.dataType, r.dataType) match {
309309
case (CalendarIntervalType, CalendarIntervalType) => s
310310
case (DateType, CalendarIntervalType) => DateAddInterval(l, UnaryMinus(r))
311-
case (_, CalendarIntervalType) => Cast(TimeSub(l, r), l.dataType)
311+
case (_, CalendarIntervalType) => Cast(TimeAdd(l, UnaryMinus(r)), l.dataType)
312312
case (TimestampType, _) => SubtractTimestamps(l, r)
313313
case (_, TimestampType) => SubtractTimestamps(l, r)
314314
case (_, DateType) => SubtractDates(l, r)

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/StreamingJoinHelper.scala

Lines changed: 0 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -232,8 +232,6 @@ object StreamingJoinHelper extends PredicateHelper with Logging {
232232
collect(left, negate) ++ collect(right, !negate)
233233
case TimeAdd(left, right, _) =>
234234
collect(left, negate) ++ collect(right, negate)
235-
case TimeSub(left, right, _) =>
236-
collect(left, negate) ++ collect(right, !negate)
237235
case UnaryMinus(child) =>
238236
collect(child, !negate)
239237
case CheckOverflow(child, _, _) =>

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/TypeCoercion.scala

Lines changed: 0 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -831,10 +831,7 @@ object TypeCoercion {
831831
case s @ SubtractTimestamps(_, DateType()) =>
832832
s.copy(startTimestamp = Cast(s.startTimestamp, TimestampType))
833833

834-
case t @ TimeAdd(DateType(), _, _) => t.copy(start = Cast(t.start, TimestampType))
835834
case t @ TimeAdd(StringType(), _, _) => t.copy(start = Cast(t.start, TimestampType))
836-
case t @ TimeSub(DateType(), _, _) => t.copy(start = Cast(t.start, TimestampType))
837-
case t @ TimeSub(StringType(), _, _) => t.copy(start = Cast(t.start, TimestampType))
838835
}
839836
}
840837

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/datetimeExpressions.scala

Lines changed: 0 additions & 35 deletions
Original file line numberDiff line numberDiff line change
@@ -1330,41 +1330,6 @@ case class FromUTCTimestamp(left: Expression, right: Expression)
13301330
}
13311331
}
13321332

1333-
/**
1334-
* Subtracts an interval from timestamp.
1335-
*/
1336-
case class TimeSub(start: Expression, interval: Expression, timeZoneId: Option[String] = None)
1337-
extends BinaryExpression with TimeZoneAwareExpression with ExpectsInputTypes {
1338-
1339-
def this(start: Expression, interval: Expression) = this(start, interval, None)
1340-
1341-
override def left: Expression = start
1342-
override def right: Expression = interval
1343-
1344-
override def toString: String = s"$left - $right"
1345-
override def sql: String = s"${left.sql} - ${right.sql}"
1346-
override def inputTypes: Seq[AbstractDataType] = Seq(TimestampType, CalendarIntervalType)
1347-
1348-
override def dataType: DataType = TimestampType
1349-
1350-
override def withTimeZone(timeZoneId: String): TimeZoneAwareExpression =
1351-
copy(timeZoneId = Option(timeZoneId))
1352-
1353-
override def nullSafeEval(start: Any, interval: Any): Any = {
1354-
val itvl = interval.asInstanceOf[CalendarInterval]
1355-
DateTimeUtils.timestampAddInterval(
1356-
start.asInstanceOf[Long], 0 - itvl.months, 0 - itvl.days, 0 - itvl.microseconds, zoneId)
1357-
}
1358-
1359-
override def doGenCode(ctx: CodegenContext, ev: ExprCode): ExprCode = {
1360-
val zid = ctx.addReferenceObj("zoneId", zoneId, classOf[ZoneId].getName)
1361-
val dtu = DateTimeUtils.getClass.getName.stripSuffix("$")
1362-
defineCodeGen(ctx, ev, (sd, i) => {
1363-
s"""$dtu.timestampAddInterval($sd, 0 - $i.months, 0 - $i.days, 0 - $i.microseconds, $zid)"""
1364-
})
1365-
}
1366-
}
1367-
13681333
/**
13691334
* Returns the date that is num_months after start_date.
13701335
*/

sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/expressions/DateExpressionsSuite.scala

Lines changed: 16 additions & 16 deletions
Original file line numberDiff line numberDiff line change
@@ -465,54 +465,54 @@ class DateExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper {
465465
sdf.setTimeZone(TimeZone.getTimeZone(zid))
466466

467467
checkEvaluation(
468-
TimeSub(
468+
TimeAdd(
469469
Literal(new Timestamp(sdf.parse("2016-03-31 10:00:00.000").getTime)),
470-
Literal(new CalendarInterval(1, 0, 0)),
470+
UnaryMinus(Literal(new CalendarInterval(1, 0, 0))),
471471
timeZoneId),
472472
DateTimeUtils.fromJavaTimestamp(
473473
new Timestamp(sdf.parse("2016-02-29 10:00:00.000").getTime)))
474474
checkEvaluation(
475-
TimeSub(
475+
TimeAdd(
476476
Literal(new Timestamp(sdf.parse("2016-03-31 10:00:00.000").getTime)),
477-
Literal(new CalendarInterval(1, 1, 0)),
477+
UnaryMinus(Literal(new CalendarInterval(1, 1, 0))),
478478
timeZoneId),
479479
DateTimeUtils.fromJavaTimestamp(
480480
new Timestamp(sdf.parse("2016-02-28 10:00:00.000").getTime)))
481481
checkEvaluation(
482-
TimeSub(
482+
TimeAdd(
483483
Literal(new Timestamp(sdf.parse("2016-03-30 00:00:01.000").getTime)),
484-
Literal(new CalendarInterval(1, 0, 2000000.toLong)),
484+
UnaryMinus(Literal(new CalendarInterval(1, 0, 2000000.toLong))),
485485
timeZoneId),
486486
DateTimeUtils.fromJavaTimestamp(
487487
new Timestamp(sdf.parse("2016-02-28 23:59:59.000").getTime)))
488488
checkEvaluation(
489-
TimeSub(
489+
TimeAdd(
490490
Literal(new Timestamp(sdf.parse("2016-03-30 00:00:01.000").getTime)),
491-
Literal(new CalendarInterval(1, 1, 2000000.toLong)),
491+
UnaryMinus(Literal(new CalendarInterval(1, 1, 2000000.toLong))),
492492
timeZoneId),
493493
DateTimeUtils.fromJavaTimestamp(
494494
new Timestamp(sdf.parse("2016-02-27 23:59:59.000").getTime)))
495495

496496
checkEvaluation(
497-
TimeSub(
497+
TimeAdd(
498498
Literal.create(null, TimestampType),
499-
Literal(new CalendarInterval(1, 2, 123000L)),
499+
UnaryMinus(Literal(new CalendarInterval(1, 2, 123000L))),
500500
timeZoneId),
501501
null)
502502
checkEvaluation(
503-
TimeSub(
503+
TimeAdd(
504504
Literal(new Timestamp(sdf.parse("2016-01-29 10:00:00.000").getTime)),
505-
Literal.create(null, CalendarIntervalType),
505+
UnaryMinus(Literal.create(null, CalendarIntervalType)),
506506
timeZoneId),
507507
null)
508508
checkEvaluation(
509-
TimeSub(
509+
TimeAdd(
510510
Literal.create(null, TimestampType),
511-
Literal.create(null, CalendarIntervalType),
511+
UnaryMinus(Literal.create(null, CalendarIntervalType)),
512512
timeZoneId),
513513
null)
514-
checkConsistencyBetweenInterpretedAndCodegen(
515-
(start: Expression, interval: Expression) => TimeSub(start, interval, timeZoneId),
514+
checkConsistencyBetweenInterpretedAndCodegen((start: Expression, interval: Expression) =>
515+
TimeAdd(start, UnaryMinus(interval), timeZoneId),
516516
TimestampType, CalendarIntervalType)
517517
}
518518
}

sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/expressions/ExpressionSQLBuilderSuite.scala

Lines changed: 0 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -172,10 +172,6 @@ class ExpressionSQLBuilderSuite extends SparkFunSuite {
172172
"`a` + INTERVAL '1 hours'"
173173
)
174174

175-
checkSQL(
176-
TimeSub('a, interval),
177-
"`a` - INTERVAL '1 hours'"
178-
)
179175
checkSQL(
180176
DateAddInterval('a, interval),
181177
"`a` + INTERVAL '1 hours'"

sql/core/src/main/scala/org/apache/spark/sql/execution/streaming/StreamingSymmetricHashJoinHelper.scala

Lines changed: 1 addition & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -23,13 +23,11 @@ import org.apache.spark.{Partition, SparkContext, TaskContext}
2323
import org.apache.spark.internal.Logging
2424
import org.apache.spark.rdd.{RDD, ZippedPartitionsBaseRDD, ZippedPartitionsPartition, ZippedPartitionsRDD2}
2525
import org.apache.spark.sql.catalyst.analysis.StreamingJoinHelper
26-
import org.apache.spark.sql.catalyst.expressions.{Add, And, Attribute, AttributeReference, AttributeSet, BoundReference, Cast, CheckOverflow, Expression, ExpressionSet, GreaterThan, GreaterThanOrEqual, LessThan, LessThanOrEqual, Literal, Multiply, NamedExpression, PreciseTimestampConversion, PredicateHelper, Subtract, TimeAdd, TimeSub, UnaryMinus}
26+
import org.apache.spark.sql.catalyst.expressions.{And, Attribute, AttributeSet, BoundReference, Expression, NamedExpression, PredicateHelper}
2727
import org.apache.spark.sql.catalyst.plans.logical.EventTimeWatermark._
2828
import org.apache.spark.sql.execution.SparkPlan
2929
import org.apache.spark.sql.execution.streaming.WatermarkSupport.watermarkExpression
3030
import org.apache.spark.sql.execution.streaming.state.{StateStoreCoordinatorRef, StateStoreProvider, StateStoreProviderId}
31-
import org.apache.spark.sql.types._
32-
import org.apache.spark.unsafe.types.CalendarInterval
3331

3432

3533
/**

sql/core/src/test/resources/sql-tests/results/ansi/datetime.sql.out

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -152,7 +152,7 @@ struct<CAST(TIMESTAMP '2011-11-11 11:11:11' + INTERVAL '2 days' AS TIMESTAMP):ti
152152
-- !query
153153
select timestamp'2011-11-11 11:11:11' - interval '2' day
154154
-- !query schema
155-
struct<CAST(TIMESTAMP '2011-11-11 11:11:11' - INTERVAL '2 days' AS TIMESTAMP):timestamp>
155+
struct<CAST(TIMESTAMP '2011-11-11 11:11:11' + (- INTERVAL '2 days') AS TIMESTAMP):timestamp>
156156
-- !query output
157157
2011-11-09 11:11:11
158158

@@ -178,23 +178,23 @@ requirement failed: Cannot add hours, minutes or seconds, milliseconds, microsec
178178
-- !query
179179
select '2011-11-11' - interval '2' day
180180
-- !query schema
181-
struct<CAST(CAST(2011-11-11 AS TIMESTAMP) - INTERVAL '2 days' AS STRING):string>
181+
struct<CAST(CAST(2011-11-11 AS TIMESTAMP) + (- INTERVAL '2 days') AS STRING):string>
182182
-- !query output
183183
2011-11-09 00:00:00
184184

185185

186186
-- !query
187187
select '2011-11-11 11:11:11' - interval '2' second
188188
-- !query schema
189-
struct<CAST(CAST(2011-11-11 11:11:11 AS TIMESTAMP) - INTERVAL '2 seconds' AS STRING):string>
189+
struct<CAST(CAST(2011-11-11 11:11:11 AS TIMESTAMP) + (- INTERVAL '2 seconds') AS STRING):string>
190190
-- !query output
191191
2011-11-11 11:11:09
192192

193193

194194
-- !query
195195
select '1' - interval '2' second
196196
-- !query schema
197-
struct<CAST(CAST(1 AS TIMESTAMP) - INTERVAL '2 seconds' AS STRING):string>
197+
struct<CAST(CAST(1 AS TIMESTAMP) + (- INTERVAL '2 seconds') AS STRING):string>
198198
-- !query output
199199
NULL
200200

@@ -205,7 +205,7 @@ select 1 - interval '2' second
205205
struct<>
206206
-- !query output
207207
org.apache.spark.sql.AnalysisException
208-
cannot resolve '1 - INTERVAL '2 seconds'' due to data type mismatch: argument 1 requires timestamp type, however, '1' is of int type.; line 1 pos 7
208+
cannot resolve '1 + (- INTERVAL '2 seconds')' due to data type mismatch: argument 1 requires timestamp type, however, '1' is of int type.; line 1 pos 7
209209

210210

211211
-- !query

sql/core/src/test/resources/sql-tests/results/ansi/interval.sql.out

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -705,7 +705,7 @@ select
705705
interval '2-2' year to month + tsval
706706
from interval_arithmetic
707707
-- !query schema
708-
struct<tsval:timestamp,CAST(tsval - INTERVAL '2 years 2 months' AS TIMESTAMP):timestamp,CAST(tsval - INTERVAL '-2 years -2 months' AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '2 years 2 months' AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '-2 years -2 months' AS TIMESTAMP):timestamp,CAST(tsval + (- INTERVAL '2 years 2 months') AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '2 years 2 months' AS TIMESTAMP):timestamp>
708+
struct<tsval:timestamp,CAST(tsval + (- INTERVAL '2 years 2 months') AS TIMESTAMP):timestamp,CAST(tsval + (- INTERVAL '-2 years -2 months') AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '2 years 2 months' AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '-2 years -2 months' AS TIMESTAMP):timestamp,CAST(tsval + (- INTERVAL '2 years 2 months') AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '2 years 2 months' AS TIMESTAMP):timestamp>
709709
-- !query output
710710
2012-01-01 00:00:00 2009-11-01 00:00:00 2014-03-01 00:00:00 2014-03-01 00:00:00 2009-11-01 00:00:00 2009-11-01 00:00:00 2014-03-01 00:00:00
711711

@@ -749,7 +749,7 @@ select
749749
interval '99 11:22:33.123456789' day to second + tsval
750750
from interval_arithmetic
751751
-- !query schema
752-
struct<tsval:timestamp,CAST(tsval - INTERVAL '99 days 11 hours 22 minutes 33.123456 seconds' AS TIMESTAMP):timestamp,CAST(tsval - INTERVAL '-99 days -11 hours -22 minutes -33.123456 seconds' AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '99 days 11 hours 22 minutes 33.123456 seconds' AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '-99 days -11 hours -22 minutes -33.123456 seconds' AS TIMESTAMP):timestamp,CAST(tsval + (- INTERVAL '99 days 11 hours 22 minutes 33.123456 seconds') AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '99 days 11 hours 22 minutes 33.123456 seconds' AS TIMESTAMP):timestamp>
752+
struct<tsval:timestamp,CAST(tsval + (- INTERVAL '99 days 11 hours 22 minutes 33.123456 seconds') AS TIMESTAMP):timestamp,CAST(tsval + (- INTERVAL '-99 days -11 hours -22 minutes -33.123456 seconds') AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '99 days 11 hours 22 minutes 33.123456 seconds' AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '-99 days -11 hours -22 minutes -33.123456 seconds' AS TIMESTAMP):timestamp,CAST(tsval + (- INTERVAL '99 days 11 hours 22 minutes 33.123456 seconds') AS TIMESTAMP):timestamp,CAST(tsval + INTERVAL '99 days 11 hours 22 minutes 33.123456 seconds' AS TIMESTAMP):timestamp>
753753
-- !query output
754754
2012-01-01 00:00:00 2011-09-23 12:37:26.876544 2012-04-09 11:22:33.123456 2012-04-09 11:22:33.123456 2011-09-23 12:37:26.876544 2011-09-23 12:37:26.876544 2012-04-09 11:22:33.123456
755755

sql/core/src/test/resources/sql-tests/results/datetime.sql.out

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -126,7 +126,7 @@ struct<CAST(TIMESTAMP '2011-11-11 11:11:11' + INTERVAL '2 days' AS TIMESTAMP):ti
126126
-- !query
127127
select timestamp'2011-11-11 11:11:11' - interval '2' day
128128
-- !query schema
129-
struct<CAST(TIMESTAMP '2011-11-11 11:11:11' - INTERVAL '2 days' AS TIMESTAMP):timestamp>
129+
struct<CAST(TIMESTAMP '2011-11-11 11:11:11' + (- INTERVAL '2 days') AS TIMESTAMP):timestamp>
130130
-- !query output
131131
2011-11-09 11:11:11
132132

@@ -150,23 +150,23 @@ struct<DATE '2011-11-11' + (- INTERVAL '2 seconds'):date>
150150
-- !query
151151
select '2011-11-11' - interval '2' day
152152
-- !query schema
153-
struct<CAST(CAST(2011-11-11 AS TIMESTAMP) - INTERVAL '2 days' AS STRING):string>
153+
struct<CAST(CAST(2011-11-11 AS TIMESTAMP) + (- INTERVAL '2 days') AS STRING):string>
154154
-- !query output
155155
2011-11-09 00:00:00
156156

157157

158158
-- !query
159159
select '2011-11-11 11:11:11' - interval '2' second
160160
-- !query schema
161-
struct<CAST(CAST(2011-11-11 11:11:11 AS TIMESTAMP) - INTERVAL '2 seconds' AS STRING):string>
161+
struct<CAST(CAST(2011-11-11 11:11:11 AS TIMESTAMP) + (- INTERVAL '2 seconds') AS STRING):string>
162162
-- !query output
163163
2011-11-11 11:11:09
164164

165165

166166
-- !query
167167
select '1' - interval '2' second
168168
-- !query schema
169-
struct<CAST(CAST(1 AS TIMESTAMP) - INTERVAL '2 seconds' AS STRING):string>
169+
struct<CAST(CAST(1 AS TIMESTAMP) + (- INTERVAL '2 seconds') AS STRING):string>
170170
-- !query output
171171
NULL
172172

@@ -177,7 +177,7 @@ select 1 - interval '2' second
177177
struct<>
178178
-- !query output
179179
org.apache.spark.sql.AnalysisException
180-
cannot resolve '1 - INTERVAL '2 seconds'' due to data type mismatch: argument 1 requires timestamp type, however, '1' is of int type.; line 1 pos 7
180+
cannot resolve '1 + (- INTERVAL '2 seconds')' due to data type mismatch: argument 1 requires timestamp type, however, '1' is of int type.; line 1 pos 7
181181

182182

183183
-- !query

0 commit comments

Comments
 (0)