Skip to content

Commit 4a273a1

Browse files
committed
Revert "[SPARK-25496][SQL] Deprecate from_utc_timestamp and to_utc_timestamp"
This reverts commit 1d20d13.
1 parent 53fd83a commit 4a273a1

File tree

13 files changed

+148
-268
lines changed

13 files changed

+148
-268
lines changed

R/pkg/R/functions.R

Lines changed: 0 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -2496,7 +2496,6 @@ setMethod("schema_of_csv", signature(x = "characterOrColumn"),
24962496
#' @note from_utc_timestamp since 1.5.0
24972497
setMethod("from_utc_timestamp", signature(y = "Column", x = "character"),
24982498
function(y, x) {
2499-
.Deprecated(msg = "from_utc_timestamp is deprecated. See SPARK-25496.")
25002499
jc <- callJStatic("org.apache.spark.sql.functions", "from_utc_timestamp", y@jc, x)
25012500
column(jc)
25022501
})
@@ -2555,7 +2554,6 @@ setMethod("next_day", signature(y = "Column", x = "character"),
25552554
#' @note to_utc_timestamp since 1.5.0
25562555
setMethod("to_utc_timestamp", signature(y = "Column", x = "character"),
25572556
function(y, x) {
2558-
.Deprecated(msg = "to_utc_timestamp is deprecated. See SPARK-25496.")
25592557
jc <- callJStatic("org.apache.spark.sql.functions", "to_utc_timestamp", y@jc, x)
25602558
column(jc)
25612559
})

R/pkg/tests/fulltests/test_sparkSQL.R

Lines changed: 4 additions & 14 deletions
Original file line numberDiff line numberDiff line change
@@ -1907,20 +1907,10 @@ test_that("date functions on a DataFrame", {
19071907
df2 <- createDataFrame(l2)
19081908
expect_equal(collect(select(df2, minute(df2$b)))[, 1], c(34, 24))
19091909
expect_equal(collect(select(df2, second(df2$b)))[, 1], c(0, 34))
1910-
conf <- callJMethod(sparkSession, "conf")
1911-
isUtcTimestampFuncEnabled <- callJMethod(conf, "get", "spark.sql.legacy.utcTimestampFunc.enabled")
1912-
callJMethod(conf, "set", "spark.sql.legacy.utcTimestampFunc.enabled", "true")
1913-
tryCatch({
1914-
# Both from_utc_timestamp and to_utc_timestamp are deprecated as of SPARK-25496
1915-
expect_equal(suppressWarnings(collect(select(df2, from_utc_timestamp(df2$b, "JST"))))[, 1],
1916-
c(as.POSIXct("2012-12-13 21:34:00 UTC"), as.POSIXct("2014-12-15 10:24:34 UTC")))
1917-
expect_equal(suppressWarnings(collect(select(df2, to_utc_timestamp(df2$b, "JST"))))[, 1],
1918-
c(as.POSIXct("2012-12-13 03:34:00 UTC"), as.POSIXct("2014-12-14 16:24:34 UTC")))
1919-
},
1920-
finally = {
1921-
# Reverting the conf back
1922-
callJMethod(conf, "set", "spark.sql.legacy.utcTimestampFunc.enabled", isUtcTimestampFuncEnabled)
1923-
})
1910+
expect_equal(collect(select(df2, from_utc_timestamp(df2$b, "JST")))[, 1],
1911+
c(as.POSIXct("2012-12-13 21:34:00 UTC"), as.POSIXct("2014-12-15 10:24:34 UTC")))
1912+
expect_equal(collect(select(df2, to_utc_timestamp(df2$b, "JST")))[, 1],
1913+
c(as.POSIXct("2012-12-13 03:34:00 UTC"), as.POSIXct("2014-12-14 16:24:34 UTC")))
19241914
expect_gt(collect(select(df2, unix_timestamp()))[1, 1], 0)
19251915
expect_gt(collect(select(df2, unix_timestamp(df2$b)))[1, 1], 0)
19261916
expect_gt(collect(select(df2, unix_timestamp(lit("2015-01-01"), "yyyy-MM-dd")))[1, 1], 0)

python/pyspark/sql/functions.py

Lines changed: 0 additions & 10 deletions
Original file line numberDiff line numberDiff line change
@@ -1311,10 +1311,7 @@ def from_utc_timestamp(timestamp, tz):
13111311
[Row(local_time=datetime.datetime(1997, 2, 28, 2, 30))]
13121312
>>> df.select(from_utc_timestamp(df.ts, df.tz).alias('local_time')).collect()
13131313
[Row(local_time=datetime.datetime(1997, 2, 28, 19, 30))]
1314-
1315-
.. note:: Deprecated in 3.0. See SPARK-25496
13161314
"""
1317-
warnings.warn("Deprecated in 3.0. See SPARK-25496", DeprecationWarning)
13181315
sc = SparkContext._active_spark_context
13191316
if isinstance(tz, Column):
13201317
tz = _to_java_column(tz)
@@ -1348,10 +1345,7 @@ def to_utc_timestamp(timestamp, tz):
13481345
[Row(utc_time=datetime.datetime(1997, 2, 28, 18, 30))]
13491346
>>> df.select(to_utc_timestamp(df.ts, df.tz).alias('utc_time')).collect()
13501347
[Row(utc_time=datetime.datetime(1997, 2, 28, 1, 30))]
1351-
1352-
.. note:: Deprecated in 3.0. See SPARK-25496
13531348
"""
1354-
warnings.warn("Deprecated in 3.0. See SPARK-25496", DeprecationWarning)
13551349
sc = SparkContext._active_spark_context
13561350
if isinstance(tz, Column):
13571351
tz = _to_java_column(tz)
@@ -2955,13 +2949,9 @@ def _test():
29552949
globs['sc'] = sc
29562950
globs['spark'] = spark
29572951
globs['df'] = spark.createDataFrame([Row(name='Alice', age=2), Row(name='Bob', age=5)])
2958-
2959-
spark.conf.set("spark.sql.legacy.utcTimestampFunc.enabled", "true")
29602952
(failure_count, test_count) = doctest.testmod(
29612953
pyspark.sql.functions, globs=globs,
29622954
optionflags=doctest.ELLIPSIS | doctest.NORMALIZE_WHITESPACE)
2963-
spark.conf.unset("spark.sql.legacy.utcTimestampFunc.enabled")
2964-
29652955
spark.stop()
29662956
if failure_count:
29672957
sys.exit(-1)

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

Lines changed: 0 additions & 11 deletions
Original file line numberDiff line numberDiff line change
@@ -33,7 +33,6 @@ import org.apache.spark.sql.catalyst.expressions.codegen.Block._
3333
import org.apache.spark.sql.catalyst.util.{DateTimeUtils, TimestampFormatter}
3434
import org.apache.spark.sql.catalyst.util.DateTimeConstants._
3535
import org.apache.spark.sql.catalyst.util.DateTimeUtils._
36-
import org.apache.spark.sql.internal.SQLConf
3736
import org.apache.spark.sql.types._
3837
import org.apache.spark.unsafe.types.{CalendarInterval, UTF8String}
3938

@@ -1161,11 +1160,6 @@ case class TimeAdd(start: Expression, interval: Expression, timeZoneId: Option[S
11611160
case class FromUTCTimestamp(left: Expression, right: Expression)
11621161
extends BinaryExpression with ImplicitCastInputTypes {
11631162

1164-
if (!SQLConf.get.utcTimestampFuncEnabled) {
1165-
throw new AnalysisException(s"The $prettyName function has been disabled since Spark 3.0." +
1166-
s"Set ${SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key} to true to enable this function.")
1167-
}
1168-
11691163
override def inputTypes: Seq[AbstractDataType] = Seq(TimestampType, StringType)
11701164
override def dataType: DataType = TimestampType
11711165
override def prettyName: String = "from_utc_timestamp"
@@ -1375,11 +1369,6 @@ case class MonthsBetween(
13751369
case class ToUTCTimestamp(left: Expression, right: Expression)
13761370
extends BinaryExpression with ImplicitCastInputTypes {
13771371

1378-
if (!SQLConf.get.utcTimestampFuncEnabled) {
1379-
throw new AnalysisException(s"The $prettyName function has been disabled since Spark 3.0. " +
1380-
s"Set ${SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key} to true to enable this function.")
1381-
}
1382-
13831372
override def inputTypes: Seq[AbstractDataType] = Seq(TimestampType, StringType)
13841373
override def dataType: DataType = TimestampType
13851374
override def prettyName: String = "to_utc_timestamp"

sql/catalyst/src/main/scala/org/apache/spark/sql/internal/SQLConf.scala

Lines changed: 0 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -2038,12 +2038,6 @@ object SQLConf {
20382038
.booleanConf
20392039
.createWithDefault(false)
20402040

2041-
val UTC_TIMESTAMP_FUNC_ENABLED = buildConf("spark.sql.legacy.utcTimestampFunc.enabled")
2042-
.doc("The configuration property enables the to_utc_timestamp() " +
2043-
"and from_utc_timestamp() functions.")
2044-
.booleanConf
2045-
.createWithDefault(false)
2046-
20472041
val SOURCES_BINARY_FILE_MAX_LENGTH = buildConf("spark.sql.sources.binaryFile.maxLength")
20482042
.doc("The max length of a file that can be read by the binary file data source. " +
20492043
"Spark will fail fast and not attempt to read the file if its length exceeds this value. " +
@@ -2423,8 +2417,6 @@ class SQLConf extends Serializable with Logging {
24232417

24242418
def datetimeJava8ApiEnabled: Boolean = getConf(DATETIME_JAVA8API_ENABLED)
24252419

2426-
def utcTimestampFuncEnabled: Boolean = getConf(UTC_TIMESTAMP_FUNC_ENABLED)
2427-
24282420
def addDirectoryRecursiveEnabled: Boolean = getConf(LEGACY_ADD_DIRECTORY_USING_RECURSIVE)
24292421

24302422
def legacyMsSqlServerNumericMappingEnabled: Boolean =

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

Lines changed: 26 additions & 33 deletions
Original file line numberDiff line numberDiff line change
@@ -28,7 +28,6 @@ import org.apache.spark.sql.catalyst.expressions.codegen._
2828
import org.apache.spark.sql.catalyst.expressions.codegen.Block._
2929
import org.apache.spark.sql.catalyst.expressions.objects._
3030
import org.apache.spark.sql.catalyst.util.{ArrayBasedMapData, DateTimeUtils}
31-
import org.apache.spark.sql.internal.SQLConf
3231
import org.apache.spark.sql.types._
3332
import org.apache.spark.unsafe.types.UTF8String
3433
import org.apache.spark.util.ThreadUtils
@@ -187,42 +186,36 @@ class CodeGenerationSuite extends SparkFunSuite with ExpressionEvalHelper {
187186
}
188187

189188
test("SPARK-17702: split wide constructor into blocks due to JVM code size limit") {
190-
withSQLConf(SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key -> "true") {
191-
val length = 5000
192-
val expressions = Seq.fill(length) {
193-
ToUTCTimestamp(
194-
Literal.create(Timestamp.valueOf("2015-07-24 00:00:00"), TimestampType),
195-
Literal.create("PST", StringType))
196-
}
197-
val plan = GenerateMutableProjection.generate(expressions)
198-
val actual = plan(new GenericInternalRow(length)).toSeq(expressions.map(_.dataType))
199-
val expected = Seq.fill(length)(
200-
DateTimeUtils.fromJavaTimestamp(Timestamp.valueOf("2015-07-24 07:00:00")))
201-
202-
if (actual != expected) {
203-
fail(
204-
s"Incorrect Evaluation: expressions: $expressions, actual: $actual, expected: $expected")
205-
}
189+
val length = 5000
190+
val expressions = Seq.fill(length) {
191+
ToUTCTimestamp(
192+
Literal.create(Timestamp.valueOf("2015-07-24 00:00:00"), TimestampType),
193+
Literal.create("PST", StringType))
194+
}
195+
val plan = GenerateMutableProjection.generate(expressions)
196+
val actual = plan(new GenericInternalRow(length)).toSeq(expressions.map(_.dataType))
197+
val expected = Seq.fill(length)(
198+
DateTimeUtils.fromJavaTimestamp(Timestamp.valueOf("2015-07-24 07:00:00")))
199+
200+
if (actual != expected) {
201+
fail(s"Incorrect Evaluation: expressions: $expressions, actual: $actual, expected: $expected")
206202
}
207203
}
208204

209205
test("SPARK-22226: group splitted expressions into one method per nested class") {
210-
withSQLConf(SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key -> "true") {
211-
val length = 10000
212-
val expressions = Seq.fill(length) {
213-
ToUTCTimestamp(
214-
Literal.create(Timestamp.valueOf("2017-10-10 00:00:00"), TimestampType),
215-
Literal.create("PST", StringType))
216-
}
217-
val plan = GenerateMutableProjection.generate(expressions)
218-
val actual = plan(new GenericInternalRow(length)).toSeq(expressions.map(_.dataType))
219-
val expected = Seq.fill(length)(
220-
DateTimeUtils.fromJavaTimestamp(Timestamp.valueOf("2017-10-10 07:00:00")))
221-
222-
if (actual != expected) {
223-
fail(
224-
s"Incorrect Evaluation: expressions: $expressions, actual: $actual, expected: $expected")
225-
}
206+
val length = 10000
207+
val expressions = Seq.fill(length) {
208+
ToUTCTimestamp(
209+
Literal.create(Timestamp.valueOf("2017-10-10 00:00:00"), TimestampType),
210+
Literal.create("PST", StringType))
211+
}
212+
val plan = GenerateMutableProjection.generate(expressions)
213+
val actual = plan(new GenericInternalRow(length)).toSeq(expressions.map(_.dataType))
214+
val expected = Seq.fill(length)(
215+
DateTimeUtils.fromJavaTimestamp(Timestamp.valueOf("2017-10-10 07:00:00")))
216+
217+
if (actual != expected) {
218+
fail(s"Incorrect Evaluation: expressions: $expressions, actual: $actual, expected: $expected")
226219
}
227220
}
228221

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

Lines changed: 22 additions & 41 deletions
Original file line numberDiff line numberDiff line change
@@ -25,14 +25,12 @@ import java.util.concurrent.TimeUnit
2525
import java.util.concurrent.TimeUnit._
2626

2727
import org.apache.spark.SparkFunSuite
28-
import org.apache.spark.sql.AnalysisException
2928
import org.apache.spark.sql.catalyst.InternalRow
3029
import org.apache.spark.sql.catalyst.expressions.codegen.GenerateUnsafeProjection
3130
import org.apache.spark.sql.catalyst.util.{DateTimeUtils, IntervalUtils, TimestampFormatter}
3231
import org.apache.spark.sql.catalyst.util.DateTimeConstants.NANOS_PER_SECOND
3332
import org.apache.spark.sql.catalyst.util.DateTimeTestUtils._
3433
import org.apache.spark.sql.catalyst.util.DateTimeUtils.TimeZoneGMT
35-
import org.apache.spark.sql.internal.SQLConf
3634
import org.apache.spark.sql.types._
3735
import org.apache.spark.unsafe.types.{CalendarInterval, UTF8String}
3836

@@ -894,29 +892,21 @@ class DateExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper {
894892
NonFoldableLiteral.create(tz, StringType)),
895893
if (expected != null) Timestamp.valueOf(expected) else null)
896894
}
897-
withSQLConf(SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key -> "true") {
898-
test("2015-07-24 00:00:00", "PST", "2015-07-24 07:00:00")
899-
test("2015-01-24 00:00:00", "PST", "2015-01-24 08:00:00")
900-
test(null, "UTC", null)
901-
test("2015-07-24 00:00:00", null, null)
902-
test(null, null, null)
903-
}
904-
val msg = intercept[AnalysisException] {
905-
test("2015-07-24 00:00:00", "PST", "2015-07-24 07:00:00")
906-
}.getMessage
907-
assert(msg.contains(SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key))
895+
test("2015-07-24 00:00:00", "PST", "2015-07-24 07:00:00")
896+
test("2015-01-24 00:00:00", "PST", "2015-01-24 08:00:00")
897+
test(null, "UTC", null)
898+
test("2015-07-24 00:00:00", null, null)
899+
test(null, null, null)
908900
}
909901

910902
test("to_utc_timestamp - invalid time zone id") {
911-
withSQLConf(SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key -> "true") {
912-
Seq("Invalid time zone", "\"quote", "UTC*42").foreach { invalidTz =>
913-
val msg = intercept[java.time.DateTimeException] {
914-
GenerateUnsafeProjection.generate(
915-
ToUTCTimestamp(
916-
Literal(Timestamp.valueOf("2015-07-24 00:00:00")), Literal(invalidTz)) :: Nil)
917-
}.getMessage
918-
assert(msg.contains(invalidTz))
919-
}
903+
Seq("Invalid time zone", "\"quote", "UTC*42").foreach { invalidTz =>
904+
val msg = intercept[java.time.DateTimeException] {
905+
GenerateUnsafeProjection.generate(
906+
ToUTCTimestamp(
907+
Literal(Timestamp.valueOf("2015-07-24 00:00:00")), Literal(invalidTz)) :: Nil)
908+
}.getMessage
909+
assert(msg.contains(invalidTz))
920910
}
921911
}
922912

@@ -933,28 +923,19 @@ class DateExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper {
933923
NonFoldableLiteral.create(tz, StringType)),
934924
if (expected != null) Timestamp.valueOf(expected) else null)
935925
}
936-
withSQLConf(SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key -> "true") {
937-
test("2015-07-24 00:00:00", "PST", "2015-07-23 17:00:00")
938-
test("2015-01-24 00:00:00", "PST", "2015-01-23 16:00:00")
939-
test(null, "UTC", null)
940-
test("2015-07-24 00:00:00", null, null)
941-
test(null, null, null)
942-
}
943-
val msg = intercept[AnalysisException] {
944-
test("2015-07-24 00:00:00", "PST", "2015-07-23 17:00:00")
945-
}.getMessage
946-
assert(msg.contains(SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key))
926+
test("2015-07-24 00:00:00", "PST", "2015-07-23 17:00:00")
927+
test("2015-01-24 00:00:00", "PST", "2015-01-23 16:00:00")
928+
test(null, "UTC", null)
929+
test("2015-07-24 00:00:00", null, null)
930+
test(null, null, null)
947931
}
948932

949933
test("from_utc_timestamp - invalid time zone id") {
950-
withSQLConf(SQLConf.UTC_TIMESTAMP_FUNC_ENABLED.key -> "true") {
951-
Seq("Invalid time zone", "\"quote", "UTC*42").foreach { invalidTz =>
952-
val msg = intercept[java.time.DateTimeException] {
953-
GenerateUnsafeProjection.generate(
954-
FromUTCTimestamp(Literal(0), Literal(invalidTz)) :: Nil)
955-
}.getMessage
956-
assert(msg.contains(invalidTz))
957-
}
934+
Seq("Invalid time zone", "\"quote", "UTC*42").foreach { invalidTz =>
935+
val msg = intercept[java.time.DateTimeException] {
936+
GenerateUnsafeProjection.generate(FromUTCTimestamp(Literal(0), Literal(invalidTz)) :: Nil)
937+
}.getMessage
938+
assert(msg.contains(invalidTz))
958939
}
959940
}
960941

sql/core/src/main/scala/org/apache/spark/sql/functions.scala

Lines changed: 0 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -3053,7 +3053,6 @@ object functions {
30533053
* @group datetime_funcs
30543054
* @since 1.5.0
30553055
*/
3056-
@deprecated("This function is deprecated and will be removed in future versions.", "3.0.0")
30573056
def from_utc_timestamp(ts: Column, tz: String): Column = withExpr {
30583057
FromUTCTimestamp(ts.expr, Literal(tz))
30593058
}
@@ -3065,7 +3064,6 @@ object functions {
30653064
* @group datetime_funcs
30663065
* @since 2.4.0
30673066
*/
3068-
@deprecated("This function is deprecated and will be removed in future versions.", "3.0.0")
30693067
def from_utc_timestamp(ts: Column, tz: Column): Column = withExpr {
30703068
FromUTCTimestamp(ts.expr, tz.expr)
30713069
}
@@ -3084,7 +3082,6 @@ object functions {
30843082
* @group datetime_funcs
30853083
* @since 1.5.0
30863084
*/
3087-
@deprecated("This function is deprecated and will be removed in future versions.", "3.0.0")
30883085
def to_utc_timestamp(ts: Column, tz: String): Column = withExpr {
30893086
ToUTCTimestamp(ts.expr, Literal(tz))
30903087
}
@@ -3096,7 +3093,6 @@ object functions {
30963093
* @group datetime_funcs
30973094
* @since 2.4.0
30983095
*/
3099-
@deprecated("This function is deprecated and will be removed in future versions.", "3.0.0")
31003096
def to_utc_timestamp(ts: Column, tz: Column): Column = withExpr {
31013097
ToUTCTimestamp(ts.expr, tz.expr)
31023098
}

0 commit comments

Comments
 (0)