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 @@ -430,6 +430,9 @@ object FunctionRegistry {
expression[SecondsToTimestamp]("timestamp_seconds"),
expression[MillisToTimestamp]("timestamp_millis"),
expression[MicrosToTimestamp]("timestamp_micros"),
expression[UnixSeconds]("unix_seconds"),
expression[UnixMillis]("unix_millis"),
expression[UnixMicros]("unix_micros"),

// collection functions
expression[CreateArray]("array"),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -524,6 +524,79 @@ case class MicrosToTimestamp(child: Expression)
override def prettyName: String = "timestamp_micros"
}

abstract class TimestampToLongBase extends UnaryExpression
with ExpectsInputTypes with NullIntolerant {

protected def scaleFactor: Long

override def inputTypes: Seq[AbstractDataType] = Seq(TimestampType)

override def dataType: DataType = LongType

override def nullSafeEval(input: Any): Any = {
Math.floorDiv(input.asInstanceOf[Number].longValue(), scaleFactor)
}

override protected def doGenCode(ctx: CodegenContext, ev: ExprCode): ExprCode = {
if (scaleFactor == 1) {
defineCodeGen(ctx, ev, c => c)
} else {
defineCodeGen(ctx, ev, c => s"java.lang.Math.floorDiv($c, ${scaleFactor}L)")
}
}
}

// scalastyle:off line.size.limit
@ExpressionDescription(
usage = "_FUNC_(timestamp) - Returns the number of seconds since 1970-01-01 00:00:00 UTC. Truncates higher levels of precision.",
examples = """
Examples:
> SELECT _FUNC_(TIMESTAMP('1970-01-01 00:00:01Z'));
1
""",
group = "datetime_funcs",
since = "3.1.0")
// scalastyle:on line.size.limit
case class UnixSeconds(child: Expression) extends TimestampToLongBase {
override def scaleFactor: Long = MICROS_PER_SECOND

override def prettyName: String = "unix_seconds"
}

// scalastyle:off line.size.limit
@ExpressionDescription(
usage = "_FUNC_(timestamp) - Returns the number of milliseconds since 1970-01-01 00:00:00 UTC. Truncates higher levels of precision.",
examples = """
Examples:
> SELECT _FUNC_(TIMESTAMP('1970-01-01 00:00:01Z'));
1000
""",
group = "datetime_funcs",
since = "3.1.0")
// scalastyle:on line.size.limit
case class UnixMillis(child: Expression) extends TimestampToLongBase {
override def scaleFactor: Long = MICROS_PER_MILLIS

override def prettyName: String = "unix_millis"
}

// scalastyle:off line.size.limit
@ExpressionDescription(
usage = "_FUNC_(timestamp) - Returns the number of microseconds since 1970-01-01 00:00:00 UTC.",
examples = """
Examples:
> SELECT _FUNC_(TIMESTAMP('1970-01-01 00:00:01Z'));
1000000
""",
group = "datetime_funcs",
since = "3.1.0")
// scalastyle:on line.size.limit
case class UnixMicros(child: Expression) extends TimestampToLongBase {
override def scaleFactor: Long = 1L

override def prettyName: String = "unix_micros"
}

@ExpressionDescription(
usage = "_FUNC_(date) - Returns the year component of the date/timestamp.",
examples = """
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1197,6 +1197,51 @@ class DateExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper {
checkResult(Int.MinValue.toLong - 100)
}

test("UNIX_SECONDS") {
Copy link
Contributor

@cloud-fan cloud-fan Dec 2, 2020

Choose a reason for hiding this comment

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

let's also test null and negative input (that truncates)

checkEvaluation(UnixSeconds(Literal(null, TimestampType)), null)
var timestamp = Literal(new Timestamp(0L))
checkEvaluation(UnixSeconds(timestamp), 0L)
timestamp = Literal(new Timestamp(1000L))
checkEvaluation(UnixSeconds(timestamp), 1L)
timestamp = Literal(new Timestamp(-1000L))
checkEvaluation(UnixSeconds(timestamp), -1L)
// -1ms is considered to be in -1st second, as 0-999ms is in 0th second.
timestamp = Literal(new Timestamp(-1L))
checkEvaluation(UnixSeconds(timestamp), -1L)
timestamp = Literal(new Timestamp(-1000L))
checkEvaluation(UnixSeconds(timestamp), -1L)
// Truncates higher levels of precision
timestamp = Literal(new Timestamp(1999L))
checkEvaluation(UnixSeconds(timestamp), 1L)
}

test("UNIX_MILLIS") {
checkEvaluation(UnixMillis(Literal(null, TimestampType)), null)
var timestamp = Literal(new Timestamp(0L))
checkEvaluation(UnixMillis(timestamp), 0L)
timestamp = Literal(new Timestamp(1000L))
checkEvaluation(UnixMillis(timestamp), 1000L)
timestamp = Literal(new Timestamp(-1000L))
checkEvaluation(UnixMillis(timestamp), -1000L)
// Truncates higher levels of precision
val timestampWithNanos = new Timestamp(1000L)
timestampWithNanos.setNanos(999999)
checkEvaluation(UnixMillis(Literal(timestampWithNanos)), 1000L)
}

test("UNIX_MICROS") {
checkEvaluation(UnixMicros(Literal(null, TimestampType)), null)
var timestamp = Literal(new Timestamp(0L))
checkEvaluation(UnixMicros(timestamp), 0L)
timestamp = Literal(new Timestamp(1000L))
checkEvaluation(UnixMicros(timestamp), 1000000L)
timestamp = Literal(new Timestamp(-1000L))
checkEvaluation(UnixMicros(timestamp), -1000000L)
val timestampWithNanos = new Timestamp(1000L)
timestampWithNanos.setNanos(1000) // 1 microsecond
checkEvaluation(UnixMicros(Literal(timestampWithNanos)), 1000001L)
}

test("TIMESTAMP_SECONDS") {
def testIntegralFunc(value: Number): Unit = {
checkEvaluation(
Expand Down
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
<!-- Automatically generated by ExpressionsSchemaSuite -->
## Summary
- Number of queries: 342
- Number of queries: 345
- Number of expressions that missing example: 13
- Expressions missing examples: bigint,binary,boolean,date,decimal,double,float,int,smallint,string,timestamp,tinyint,window
## Schema of Built-in Functions
Expand Down Expand Up @@ -289,6 +289,9 @@
| org.apache.spark.sql.catalyst.expressions.UnaryMinus | negative | SELECT negative(1) | struct<negative(1):int> |
| org.apache.spark.sql.catalyst.expressions.UnaryPositive | positive | SELECT positive(1) | struct<(+ 1):int> |
| org.apache.spark.sql.catalyst.expressions.Unhex | unhex | SELECT decode(unhex('537061726B2053514C'), 'UTF-8') | struct<decode(unhex(537061726B2053514C), UTF-8):string> |
| org.apache.spark.sql.catalyst.expressions.UnixMicros | unix_micros | SELECT unix_micros(TIMESTAMP('1970-01-01 00:00:01Z')) | struct<unix_micros(CAST(1970-01-01 00:00:01Z AS TIMESTAMP)):bigint> |
| org.apache.spark.sql.catalyst.expressions.UnixMillis | unix_millis | SELECT unix_millis(TIMESTAMP('1970-01-01 00:00:01Z')) | struct<unix_millis(CAST(1970-01-01 00:00:01Z AS TIMESTAMP)):bigint> |
| org.apache.spark.sql.catalyst.expressions.UnixSeconds | unix_seconds | SELECT unix_seconds(TIMESTAMP('1970-01-01 00:00:01Z')) | struct<unix_seconds(CAST(1970-01-01 00:00:01Z AS TIMESTAMP)):bigint> |
| org.apache.spark.sql.catalyst.expressions.UnixTimestamp | unix_timestamp | SELECT unix_timestamp() | struct<unix_timestamp(current_timestamp(), yyyy-MM-dd HH:mm:ss):bigint> |
| org.apache.spark.sql.catalyst.expressions.Upper | ucase | SELECT ucase('SparkSql') | struct<ucase(SparkSql):string> |
| org.apache.spark.sql.catalyst.expressions.Upper | upper | SELECT upper('SparkSql') | struct<upper(SparkSql):string> |
Expand Down
4 changes: 4 additions & 0 deletions sql/core/src/test/resources/sql-tests/inputs/datetime.sql
Original file line number Diff line number Diff line change
Expand Up @@ -14,6 +14,10 @@ select TIMESTAMP_MILLIS(-92233720368547758);
select TIMESTAMP_SECONDS(0.1234567);
-- truncation is OK for float/double
select TIMESTAMP_SECONDS(0.1234567d), TIMESTAMP_SECONDS(FLOAT(0.1234567));
-- UNIX_SECONDS, UNIX_MILLISECONDS and UNIX_MICROSECONDS
select UNIX_SECONDS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_SECONDS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_SECONDS(null);
select UNIX_MILLIS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_MILLIS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_MILLIS(null);
select UNIX_MICROS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_MICROS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_MICROS(null);

-- [SPARK-16836] current_date and current_timestamp literals
select current_date = current_date(), current_timestamp = current_timestamp();
Expand Down
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
-- Automatically generated by SQLQueryTestSuite
-- Number of queries: 117
-- Number of queries: 120


-- !query
Expand Down Expand Up @@ -87,6 +87,30 @@ struct<timestamp_seconds(0.1234567):timestamp,timestamp_seconds(CAST(0.1234567 A
1969-12-31 16:00:00.123456 1969-12-31 16:00:00.123456


-- !query
select UNIX_SECONDS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_SECONDS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_SECONDS(null)
-- !query schema
struct<unix_seconds(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_seconds(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_seconds(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008 1606833008 NULL


-- !query
select UNIX_MILLIS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_MILLIS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_MILLIS(null)
-- !query schema
struct<unix_millis(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_millis(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_millis(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008000 1606833008999 NULL


-- !query
select UNIX_MICROS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_MICROS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_MICROS(null)
-- !query schema
struct<unix_micros(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_micros(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_micros(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008000000 1606833008999999 NULL


-- !query
select current_date = current_date(), current_timestamp = current_timestamp()
-- !query schema
Expand Down
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
-- Automatically generated by SQLQueryTestSuite
-- Number of queries: 117
-- Number of queries: 120


-- !query
Expand Down Expand Up @@ -87,6 +87,30 @@ struct<timestamp_seconds(0.1234567):timestamp,timestamp_seconds(CAST(0.1234567 A
1969-12-31 16:00:00.123456 1969-12-31 16:00:00.123456


-- !query
select UNIX_SECONDS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_SECONDS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_SECONDS(null)
-- !query schema
struct<unix_seconds(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_seconds(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_seconds(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008 1606833008 NULL


-- !query
select UNIX_MILLIS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_MILLIS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_MILLIS(null)
-- !query schema
struct<unix_millis(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_millis(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_millis(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008000 1606833008999 NULL


-- !query
select UNIX_MICROS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_MICROS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_MICROS(null)
-- !query schema
struct<unix_micros(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_micros(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_micros(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008000000 1606833008999999 NULL


-- !query
select current_date = current_date(), current_timestamp = current_timestamp()
-- !query schema
Expand Down
26 changes: 25 additions & 1 deletion sql/core/src/test/resources/sql-tests/results/datetime.sql.out
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
-- Automatically generated by SQLQueryTestSuite
-- Number of queries: 117
-- Number of queries: 120


-- !query
Expand Down Expand Up @@ -87,6 +87,30 @@ struct<timestamp_seconds(0.1234567):timestamp,timestamp_seconds(CAST(0.1234567 A
1969-12-31 16:00:00.123456 1969-12-31 16:00:00.123456


-- !query
select UNIX_SECONDS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_SECONDS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_SECONDS(null)
-- !query schema
struct<unix_seconds(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_seconds(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_seconds(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008 1606833008 NULL


-- !query
select UNIX_MILLIS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_MILLIS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_MILLIS(null)
-- !query schema
struct<unix_millis(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_millis(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_millis(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008000 1606833008999 NULL


-- !query
select UNIX_MICROS(TIMESTAMP('2020-12-01 14:30:08Z')), UNIX_MICROS(TIMESTAMP('2020-12-01 14:30:08.999999Z')), UNIX_MICROS(null)
-- !query schema
struct<unix_micros(CAST(2020-12-01 14:30:08Z AS TIMESTAMP)):bigint,unix_micros(CAST(2020-12-01 14:30:08.999999Z AS TIMESTAMP)):bigint,unix_micros(CAST(NULL AS TIMESTAMP)):bigint>
-- !query output
1606833008000000 1606833008999999 NULL


-- !query
select current_date = current_date(), current_timestamp = current_timestamp()
-- !query schema
Expand Down