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
8 changes: 4 additions & 4 deletions R/pkg/tests/fulltests/test_sparkSQL.R
Original file line number Diff line number Diff line change
Expand Up @@ -1690,9 +1690,9 @@ test_that("column functions", {

df <- as.DataFrame(list(list("col" = "1")))
c <- collect(select(df, schema_of_csv("Amsterdam,2018")))
expect_equal(c[[1]], "STRUCT<`_c0`: STRING, `_c1`: INT>")
expect_equal(c[[1]], "STRUCT<_c0: STRING, _c1: INT>")
c <- collect(select(df, schema_of_csv(lit("Amsterdam,2018"))))
expect_equal(c[[1]], "STRUCT<`_c0`: STRING, `_c1`: INT>")
expect_equal(c[[1]], "STRUCT<_c0: STRING, _c1: INT>")

# Test to_json(), from_json(), schema_of_json()
df <- sql("SELECT array(named_struct('name', 'Bob'), named_struct('name', 'Alice')) as people")
Expand Down Expand Up @@ -1725,9 +1725,9 @@ test_that("column functions", {

df <- as.DataFrame(list(list("col" = "1")))
c <- collect(select(df, schema_of_json('{"name":"Bob"}')))
expect_equal(c[[1]], "STRUCT<`name`: STRING>")
expect_equal(c[[1]], "STRUCT<name: STRING>")
c <- collect(select(df, schema_of_json(lit('{"name":"Bob"}'))))
expect_equal(c[[1]], "STRUCT<`name`: STRING>")
expect_equal(c[[1]], "STRUCT<name: STRING>")

# Test to_json() supports arrays of primitive types and arrays
df <- sql("SELECT array(19, 42, 70) as age")
Expand Down
8 changes: 4 additions & 4 deletions python/pyspark/sql/functions.py
Original file line number Diff line number Diff line change
Expand Up @@ -4091,10 +4091,10 @@ def schema_of_json(json: "ColumnOrName", options: Optional[Dict[str, str]] = Non
--------
>>> df = spark.range(1)
>>> df.select(schema_of_json(lit('{"a": 0}')).alias("json")).collect()
[Row(json='STRUCT<`a`: BIGINT>')]
[Row(json='STRUCT<a: BIGINT>')]
>>> schema = schema_of_json('{a: 1}', {'allowUnquotedFieldNames':'true'})
>>> df.select(schema.alias("json")).collect()
[Row(json='STRUCT<`a`: BIGINT>')]
[Row(json='STRUCT<a: BIGINT>')]
"""
if isinstance(json, str):
col = _create_column_from_literal(json)
Expand Down Expand Up @@ -4127,9 +4127,9 @@ def schema_of_csv(csv: "ColumnOrName", options: Optional[Dict[str, str]] = None)
--------
>>> df = spark.range(1)
>>> df.select(schema_of_csv(lit('1|a'), {'sep':'|'}).alias("csv")).collect()
[Row(csv='STRUCT<`_c0`: INT, `_c1`: STRING>')]
[Row(csv='STRUCT<_c0: INT, _c1: STRING>')]
>>> df.select(schema_of_csv('1|a', {'sep':'|'}).alias("csv")).collect()
[Row(csv='STRUCT<`_c0`: INT, `_c1`: STRING>')]
[Row(csv='STRUCT<_c0: INT, _c1: STRING>')]
"""
if isinstance(csv, str):
col = _create_column_from_literal(csv)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -153,7 +153,7 @@ case class CsvToStructs(
examples = """
Examples:
> SELECT _FUNC_('1,abc');
STRUCT<`_c0`: INT, `_c1`: STRING>
STRUCT<_c0: INT, _c1: STRING>
""",
since = "3.0.0",
group = "csv_funcs")
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -766,9 +766,9 @@ case class StructsToJson(
examples = """
Examples:
> SELECT _FUNC_('[{"col":0}]');
ARRAY<STRUCT<`col`: BIGINT>>
ARRAY<STRUCT<col: BIGINT>>
> SELECT _FUNC_('[{"col":01}]', map('allowNumericLeadingZeros', 'true'));
ARRAY<STRUCT<`col`: BIGINT>>
ARRAY<STRUCT<col: BIGINT>>
""",
group = "json_funcs",
since = "2.4.0")
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,7 +21,7 @@ import org.json4s.JsonAST.JValue
import org.json4s.JsonDSL._

import org.apache.spark.annotation.Stable
import org.apache.spark.sql.catalyst.util.{escapeSingleQuotedString, quoteIdentifier}
import org.apache.spark.sql.catalyst.util.{escapeSingleQuotedString, quoteIfNeeded}
import org.apache.spark.sql.catalyst.util.StringUtils.StringConcat
import org.apache.spark.sql.util.SchemaUtils

Expand Down Expand Up @@ -93,7 +93,7 @@ case class StructField(
* Returns a string containing a schema in SQL format. For example the following value:
* `StructField("eventId", IntegerType)` will be converted to `eventId`: INT.
*/
private[sql] def sql = s"${quoteIdentifier(name)}: ${dataType.sql}$getDDLComment"
private[sql] def sql = s"${quoteIfNeeded(name)}: ${dataType.sql}$getDDLComment"

/**
* Returns a string containing a schema in DDL format. For example, the following value:
Expand All @@ -103,6 +103,6 @@ case class StructField(
*/
def toDDL: String = {
val nullString = if (nullable) "" else " NOT NULL"
s"${quoteIdentifier(name)} ${dataType.sql}${nullString}$getDDLComment"
s"${quoteIfNeeded(name)} ${dataType.sql}${nullString}$getDDLComment"
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -158,13 +158,13 @@ class CsvExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper with P
}

test("infer schema of CSV strings") {
checkEvaluation(new SchemaOfCsv(Literal.create("1,abc")), "STRUCT<`_c0`: INT, `_c1`: STRING>")
checkEvaluation(new SchemaOfCsv(Literal.create("1,abc")), "STRUCT<_c0: INT, _c1: STRING>")
}

test("infer schema of CSV strings by using options") {
checkEvaluation(
new SchemaOfCsv(Literal.create("1|abc"), Map("delimiter" -> "|")),
"STRUCT<`_c0`: INT, `_c1`: STRING>")
"STRUCT<_c0: INT, _c1: STRING>")
}

test("to_csv - struct") {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -736,17 +736,17 @@ class JsonExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper with

test("SPARK-24709: infer schema of json strings") {
checkEvaluation(new SchemaOfJson(Literal.create("""{"col":0}""")),
"STRUCT<`col`: BIGINT>")
"STRUCT<col: BIGINT>")
checkEvaluation(
new SchemaOfJson(Literal.create("""{"col0":["a"], "col1": {"col2": "b"}}""")),
"STRUCT<`col0`: ARRAY<STRING>, `col1`: STRUCT<`col2`: STRING>>")
"STRUCT<col0: ARRAY<STRING>, col1: STRUCT<col2: STRING>>")
}

test("infer schema of JSON strings by using options") {
checkEvaluation(
new SchemaOfJson(Literal.create("""{"col":01}"""),
CreateMap(Seq(Literal.create("allowNumericLeadingZeros"), Literal.create("true")))),
"STRUCT<`col`: BIGINT>")
"STRUCT<col: BIGINT>")
}

test("parse date with locale") {
Expand Down Expand Up @@ -811,7 +811,7 @@ class JsonExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper with
}

Seq("en-US", "ko-KR", "ru-RU", "de-DE").foreach {
checkDecimalInfer(_, """STRUCT<`d`: DECIMAL(7,3)>""")
checkDecimalInfer(_, """STRUCT<d: DECIMAL(7,3)>""")
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -51,7 +51,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
test("SPARK-24849: toDDL - simple struct") {
val struct = StructType(Seq(StructField("a", IntegerType)))

assert(struct.toDDL == "`a` INT")
assert(struct.toDDL == "a INT")
}

test("SPARK-24849: round trip toDDL - fromDDL") {
Expand All @@ -61,7 +61,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
}

test("SPARK-24849: round trip fromDDL - toDDL") {
val struct = "`a` MAP<INT, STRING>,`b` INT"
val struct = "a MAP<INT, STRING>,b INT"

assert(fromDDL(struct).toDDL === struct)
}
Expand All @@ -70,14 +70,14 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
val struct = new StructType()
.add("metaData", new StructType().add("eventId", StringType))

assert(struct.toDDL == "`metaData` STRUCT<`eventId`: STRING>")
assert(struct.toDDL == "metaData STRUCT<eventId: STRING>")
}

test("SPARK-24849: toDDL should output field's comment") {
val struct = StructType(Seq(
StructField("b", BooleanType).withComment("Field's comment")))

assert(struct.toDDL == """`b` BOOLEAN COMMENT 'Field\'s comment'""")
assert(struct.toDDL == """b BOOLEAN COMMENT 'Field\'s comment'""")
}

private val nestedStruct = new StructType()
Expand All @@ -89,7 +89,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
).withComment("comment"))

test("SPARK-33846: toDDL should output nested field's comment") {
val ddl = "`a` STRUCT<`b`: STRUCT<`c`: STRING COMMENT 'Deep Nested comment'> " +
val ddl = "a STRUCT<b: STRUCT<c: STRING COMMENT 'Deep Nested comment'> " +
"COMMENT 'Nested comment'> COMMENT 'comment'"
assert(nestedStruct.toDDL == ddl)
}
Expand Down Expand Up @@ -153,7 +153,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
}

test("interval keyword in schema string") {
val interval = "`a` INTERVAL"
val interval = "a INTERVAL"
assert(fromDDL(interval).toDDL === interval)
}

Expand Down Expand Up @@ -250,10 +250,10 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
}

test("SPARK-35285: ANSI interval types in schema") {
val yearMonthInterval = "`ymi` INTERVAL YEAR TO MONTH"
val yearMonthInterval = "ymi INTERVAL YEAR TO MONTH"
assert(fromDDL(yearMonthInterval).toDDL === yearMonthInterval)

val dayTimeInterval = "`dti` INTERVAL DAY TO SECOND"
val dayTimeInterval = "dti INTERVAL DAY TO SECOND"
assert(fromDDL(dayTimeInterval).toDDL === dayTimeInterval)
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -52,8 +52,8 @@ show create table char_tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.char_tbl (
`c` CHAR(5),
`v` VARCHAR(6))
c CHAR(5),
v VARCHAR(6))
USING parquet


Expand All @@ -71,8 +71,8 @@ show create table char_tbl2
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.char_tbl2 (
`c` CHAR(5),
`v` VARCHAR(6))
c CHAR(5),
v VARCHAR(6))
USING parquet


Expand Down Expand Up @@ -162,8 +162,8 @@ show create table char_tbl3
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.char_tbl3 (
`c` CHAR(5),
`v` VARCHAR(6))
c CHAR(5),
v VARCHAR(6))
Copy link
Contributor

Choose a reason for hiding this comment

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

I noticed that the view test in this file still has quotes. ShowCreateTableCommand has a branch for view, and we need to update the quoting logic in that branch as well. @Peng-Lei

Copy link
Contributor Author

Choose a reason for hiding this comment

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

Sorry, I left this out. I will do it by a follow up.

USING parquet


Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -89,7 +89,7 @@ select schema_of_csv('1|abc', map('delimiter', '|'))
-- !query schema
struct<schema_of_csv(1|abc):string>
-- !query output
STRUCT<`_c0`: INT, `_c1`: STRING>
STRUCT<_c0: INT, _c1: STRING>


-- !query
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -236,7 +236,7 @@ select schema_of_json('{"c1":0, "c2":[1]}')
-- !query schema
struct<schema_of_json({"c1":0, "c2":[1]}):string>
-- !query output
STRUCT<`c1`: BIGINT, `c2`: ARRAY<BIGINT>>
STRUCT<c1: BIGINT, c2: ARRAY<BIGINT>>


-- !query
Expand Down Expand Up @@ -375,15 +375,15 @@ select schema_of_json('{"c1":1}', map('primitivesAsString', 'true'))
-- !query schema
struct<schema_of_json({"c1":1}):string>
-- !query output
STRUCT<`c1`: STRING>
STRUCT<c1: STRING>


-- !query
select schema_of_json('{"c1":01, "c2":0.1}', map('allowNumericLeadingZeros', 'true', 'prefersDecimal', 'true'))
-- !query schema
struct<schema_of_json({"c1":01, "c2":0.1}):string>
-- !query output
STRUCT<`c1`: BIGINT, `c2`: DECIMAL(1,1)>
STRUCT<c1: BIGINT, c2: DECIMAL(1,1)>


-- !query
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -16,9 +16,9 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`a` INT,
`b` STRING,
`c` INT)
a INT,
b STRING,
c INT)
USING parquet


Expand All @@ -45,9 +45,9 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`a` INT,
`b` STRING,
`c` INT)
a INT,
b STRING,
c INT)
USING parquet
OPTIONS (
'a' = '1')
Expand Down Expand Up @@ -76,9 +76,9 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`a` INT,
`b` STRING,
`c` INT)
a INT,
b STRING,
c INT)
USING parquet
LOCATION 'file:/path/to/table'

Expand Down Expand Up @@ -106,9 +106,9 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`a` INT,
`b` STRING,
`c` INT)
a INT,
b STRING,
c INT)
USING parquet
LOCATION 'file:/path/to/table'

Expand Down Expand Up @@ -136,9 +136,9 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`b` STRING,
`c` INT,
`a` INT)
b STRING,
c INT,
a INT)
USING parquet
PARTITIONED BY (a)

Expand Down Expand Up @@ -166,9 +166,9 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`a` INT,
`b` STRING,
`c` INT)
a INT,
b STRING,
c INT)
USING parquet
CLUSTERED BY (a)
SORTED BY (b)
Expand Down Expand Up @@ -198,9 +198,9 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`a` INT,
`b` STRING,
`c` INT)
a INT,
b STRING,
c INT)
USING parquet
COMMENT 'This is a comment'

Expand Down Expand Up @@ -228,9 +228,9 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`a` INT,
`b` STRING,
`c` INT)
a INT,
b STRING,
c INT)
USING parquet
TBLPROPERTIES (
'a' = '1')
Expand Down Expand Up @@ -258,10 +258,10 @@ SHOW CREATE TABLE tbl
struct<createtab_stmt:string>
-- !query output
CREATE TABLE default.tbl (
`a` FLOAT,
`b` DECIMAL(10,0),
`c` DECIMAL(10,0),
`d` DECIMAL(10,1))
a FLOAT,
b DECIMAL(10,0),
c DECIMAL(10,0),
d DECIMAL(10,1))
USING parquet


Expand Down
Loading