Skip to content

Commit c3712b7

Browse files
committed
[SPARK-21307][REVERT][SQL] Remove SQLConf parameters from the parser-related classes
## What changes were proposed in this pull request? Since we do not set active sessions when parsing the plan, we are unable to correctly use SQLConf.get to find the correct active session. Since #18531 breaks the build, I plan to revert it at first. ## How was this patch tested? The existing test cases Author: Xiao Li <[email protected]> Closes #18568 from gatorsmile/revert18531.
1 parent 062c336 commit c3712b7

File tree

11 files changed

+127
-121
lines changed

11 files changed

+127
-121
lines changed

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/catalog/SessionCatalog.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -74,7 +74,7 @@ class SessionCatalog(
7474
functionRegistry,
7575
conf,
7676
new Configuration(),
77-
CatalystSqlParser,
77+
new CatalystSqlParser(conf),
7878
DummyFunctionResourceLoader)
7979
}
8080

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/parser/AstBuilder.scala

Lines changed: 4 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -45,9 +45,11 @@ import org.apache.spark.util.random.RandomSampler
4545
* The AstBuilder converts an ANTLR4 ParseTree into a catalyst Expression, LogicalPlan or
4646
* TableIdentifier.
4747
*/
48-
class AstBuilder extends SqlBaseBaseVisitor[AnyRef] with Logging {
48+
class AstBuilder(conf: SQLConf) extends SqlBaseBaseVisitor[AnyRef] with Logging {
4949
import ParserUtils._
5050

51+
def this() = this(new SQLConf())
52+
5153
protected def typedVisit[T](ctx: ParseTree): T = {
5254
ctx.accept(this).asInstanceOf[T]
5355
}
@@ -1457,7 +1459,7 @@ class AstBuilder extends SqlBaseBaseVisitor[AnyRef] with Logging {
14571459
* Special characters can be escaped by using Hive/C-style escaping.
14581460
*/
14591461
private def createString(ctx: StringLiteralContext): String = {
1460-
if (SQLConf.get.escapedStringLiterals) {
1462+
if (conf.escapedStringLiterals) {
14611463
ctx.STRING().asScala.map(stringWithoutUnescape).mkString
14621464
} else {
14631465
ctx.STRING().asScala.map(string).mkString

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/parser/ParseDriver.scala

Lines changed: 7 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -26,6 +26,7 @@ import org.apache.spark.sql.catalyst.{FunctionIdentifier, TableIdentifier}
2626
import org.apache.spark.sql.catalyst.expressions.Expression
2727
import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
2828
import org.apache.spark.sql.catalyst.trees.Origin
29+
import org.apache.spark.sql.internal.SQLConf
2930
import org.apache.spark.sql.types.{DataType, StructType}
3031

3132
/**
@@ -121,8 +122,13 @@ abstract class AbstractSqlParser extends ParserInterface with Logging {
121122
/**
122123
* Concrete SQL parser for Catalyst-only SQL statements.
123124
*/
125+
class CatalystSqlParser(conf: SQLConf) extends AbstractSqlParser {
126+
val astBuilder = new AstBuilder(conf)
127+
}
128+
129+
/** For test-only. */
124130
object CatalystSqlParser extends AbstractSqlParser {
125-
val astBuilder = new AstBuilder
131+
val astBuilder = new AstBuilder(new SQLConf())
126132
}
127133

128134
/**

sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/parser/ExpressionParserSuite.scala

Lines changed: 84 additions & 83 deletions
Original file line numberDiff line numberDiff line change
@@ -167,12 +167,12 @@ class ExpressionParserSuite extends PlanTest {
167167
}
168168

169169
test("like expressions with ESCAPED_STRING_LITERALS = true") {
170-
val parser = CatalystSqlParser
171-
withSQLConf(SQLConf.ESCAPED_STRING_LITERALS.key -> "true") {
172-
assertEqual("a rlike '^\\x20[\\x20-\\x23]+$'", 'a rlike "^\\x20[\\x20-\\x23]+$", parser)
173-
assertEqual("a rlike 'pattern\\\\'", 'a rlike "pattern\\\\", parser)
174-
assertEqual("a rlike 'pattern\\t\\n'", 'a rlike "pattern\\t\\n", parser)
175-
}
170+
val conf = new SQLConf()
171+
conf.setConfString(SQLConf.ESCAPED_STRING_LITERALS.key, "true")
172+
val parser = new CatalystSqlParser(conf)
173+
assertEqual("a rlike '^\\x20[\\x20-\\x23]+$'", 'a rlike "^\\x20[\\x20-\\x23]+$", parser)
174+
assertEqual("a rlike 'pattern\\\\'", 'a rlike "pattern\\\\", parser)
175+
assertEqual("a rlike 'pattern\\t\\n'", 'a rlike "pattern\\t\\n", parser)
176176
}
177177

178178
test("is null expressions") {
@@ -435,85 +435,86 @@ class ExpressionParserSuite extends PlanTest {
435435
}
436436

437437
test("strings") {
438-
val parser = CatalystSqlParser
439438
Seq(true, false).foreach { escape =>
440-
withSQLConf(SQLConf.ESCAPED_STRING_LITERALS.key -> escape.toString) {
441-
// tests that have same result whatever the conf is
442-
// Single Strings.
443-
assertEqual("\"hello\"", "hello", parser)
444-
assertEqual("'hello'", "hello", parser)
445-
446-
// Multi-Strings.
447-
assertEqual("\"hello\" 'world'", "helloworld", parser)
448-
assertEqual("'hello' \" \" 'world'", "hello world", parser)
449-
450-
// 'LIKE' string literals. Notice that an escaped '%' is the same as an escaped '\' and a
451-
// regular '%'; to get the correct result you need to add another escaped '\'.
452-
// TODO figure out if we shouldn't change the ParseUtils.unescapeSQLString method?
453-
assertEqual("'pattern%'", "pattern%", parser)
454-
assertEqual("'no-pattern\\%'", "no-pattern\\%", parser)
455-
456-
// tests that have different result regarding the conf
457-
if (escape) {
458-
// When SQLConf.ESCAPED_STRING_LITERALS is enabled, string literal parsing fallbacks to
459-
// Spark 1.6 behavior.
460-
461-
// 'LIKE' string literals.
462-
assertEqual("'pattern\\\\%'", "pattern\\\\%", parser)
463-
assertEqual("'pattern\\\\\\%'", "pattern\\\\\\%", parser)
464-
465-
// Escaped characters.
466-
// Unescape string literal "'\\0'" for ASCII NUL (X'00') doesn't work
467-
// when ESCAPED_STRING_LITERALS is enabled.
468-
// It is parsed literally.
469-
assertEqual("'\\0'", "\\0", parser)
470-
471-
// Note: Single quote follows 1.6 parsing behavior when ESCAPED_STRING_LITERALS is
472-
// enabled.
473-
val e = intercept[ParseException](parser.parseExpression("'\''"))
474-
assert(e.message.contains("extraneous input '''"))
475-
476-
// The unescape special characters (e.g., "\\t") for 2.0+ don't work
477-
// when ESCAPED_STRING_LITERALS is enabled. They are parsed literally.
478-
assertEqual("'\\\"'", "\\\"", parser) // Double quote
479-
assertEqual("'\\b'", "\\b", parser) // Backspace
480-
assertEqual("'\\n'", "\\n", parser) // Newline
481-
assertEqual("'\\r'", "\\r", parser) // Carriage return
482-
assertEqual("'\\t'", "\\t", parser) // Tab character
483-
484-
// The unescape Octals for 2.0+ don't work when ESCAPED_STRING_LITERALS is enabled.
485-
// They are parsed literally.
486-
assertEqual("'\\110\\145\\154\\154\\157\\041'", "\\110\\145\\154\\154\\157\\041", parser)
487-
// The unescape Unicode for 2.0+ doesn't work when ESCAPED_STRING_LITERALS is enabled.
488-
// They are parsed literally.
489-
assertEqual("'\\u0057\\u006F\\u0072\\u006C\\u0064\\u0020\\u003A\\u0029'",
490-
"\\u0057\\u006F\\u0072\\u006C\\u0064\\u0020\\u003A\\u0029", parser)
491-
} else {
492-
// Default behavior
493-
494-
// 'LIKE' string literals.
495-
assertEqual("'pattern\\\\%'", "pattern\\%", parser)
496-
assertEqual("'pattern\\\\\\%'", "pattern\\\\%", parser)
497-
498-
// Escaped characters.
499-
// See: http://dev.mysql.com/doc/refman/5.7/en/string-literals.html
500-
assertEqual("'\\0'", "\u0000", parser) // ASCII NUL (X'00')
501-
assertEqual("'\\''", "\'", parser) // Single quote
502-
assertEqual("'\\\"'", "\"", parser) // Double quote
503-
assertEqual("'\\b'", "\b", parser) // Backspace
504-
assertEqual("'\\n'", "\n", parser) // Newline
505-
assertEqual("'\\r'", "\r", parser) // Carriage return
506-
assertEqual("'\\t'", "\t", parser) // Tab character
507-
assertEqual("'\\Z'", "\u001A", parser) // ASCII 26 - CTRL + Z (EOF on windows)
508-
509-
// Octals
510-
assertEqual("'\\110\\145\\154\\154\\157\\041'", "Hello!", parser)
511-
512-
// Unicode
513-
assertEqual("'\\u0057\\u006F\\u0072\\u006C\\u0064\\u0020\\u003A\\u0029'", "World :)",
514-
parser)
515-
}
439+
val conf = new SQLConf()
440+
conf.setConfString(SQLConf.ESCAPED_STRING_LITERALS.key, escape.toString)
441+
val parser = new CatalystSqlParser(conf)
442+
443+
// tests that have same result whatever the conf is
444+
// Single Strings.
445+
assertEqual("\"hello\"", "hello", parser)
446+
assertEqual("'hello'", "hello", parser)
447+
448+
// Multi-Strings.
449+
assertEqual("\"hello\" 'world'", "helloworld", parser)
450+
assertEqual("'hello' \" \" 'world'", "hello world", parser)
451+
452+
// 'LIKE' string literals. Notice that an escaped '%' is the same as an escaped '\' and a
453+
// regular '%'; to get the correct result you need to add another escaped '\'.
454+
// TODO figure out if we shouldn't change the ParseUtils.unescapeSQLString method?
455+
assertEqual("'pattern%'", "pattern%", parser)
456+
assertEqual("'no-pattern\\%'", "no-pattern\\%", parser)
457+
458+
// tests that have different result regarding the conf
459+
if (escape) {
460+
// When SQLConf.ESCAPED_STRING_LITERALS is enabled, string literal parsing fallbacks to
461+
// Spark 1.6 behavior.
462+
463+
// 'LIKE' string literals.
464+
assertEqual("'pattern\\\\%'", "pattern\\\\%", parser)
465+
assertEqual("'pattern\\\\\\%'", "pattern\\\\\\%", parser)
466+
467+
// Escaped characters.
468+
// Unescape string literal "'\\0'" for ASCII NUL (X'00') doesn't work
469+
// when ESCAPED_STRING_LITERALS is enabled.
470+
// It is parsed literally.
471+
assertEqual("'\\0'", "\\0", parser)
472+
473+
// Note: Single quote follows 1.6 parsing behavior when ESCAPED_STRING_LITERALS is enabled.
474+
val e = intercept[ParseException](parser.parseExpression("'\''"))
475+
assert(e.message.contains("extraneous input '''"))
476+
477+
// The unescape special characters (e.g., "\\t") for 2.0+ don't work
478+
// when ESCAPED_STRING_LITERALS is enabled. They are parsed literally.
479+
assertEqual("'\\\"'", "\\\"", parser) // Double quote
480+
assertEqual("'\\b'", "\\b", parser) // Backspace
481+
assertEqual("'\\n'", "\\n", parser) // Newline
482+
assertEqual("'\\r'", "\\r", parser) // Carriage return
483+
assertEqual("'\\t'", "\\t", parser) // Tab character
484+
485+
// The unescape Octals for 2.0+ don't work when ESCAPED_STRING_LITERALS is enabled.
486+
// They are parsed literally.
487+
assertEqual("'\\110\\145\\154\\154\\157\\041'", "\\110\\145\\154\\154\\157\\041", parser)
488+
// The unescape Unicode for 2.0+ doesn't work when ESCAPED_STRING_LITERALS is enabled.
489+
// They are parsed literally.
490+
assertEqual("'\\u0057\\u006F\\u0072\\u006C\\u0064\\u0020\\u003A\\u0029'",
491+
"\\u0057\\u006F\\u0072\\u006C\\u0064\\u0020\\u003A\\u0029", parser)
492+
} else {
493+
// Default behavior
494+
495+
// 'LIKE' string literals.
496+
assertEqual("'pattern\\\\%'", "pattern\\%", parser)
497+
assertEqual("'pattern\\\\\\%'", "pattern\\\\%", parser)
498+
499+
// Escaped characters.
500+
// See: http://dev.mysql.com/doc/refman/5.7/en/string-literals.html
501+
assertEqual("'\\0'", "\u0000", parser) // ASCII NUL (X'00')
502+
assertEqual("'\\''", "\'", parser) // Single quote
503+
assertEqual("'\\\"'", "\"", parser) // Double quote
504+
assertEqual("'\\b'", "\b", parser) // Backspace
505+
assertEqual("'\\n'", "\n", parser) // Newline
506+
assertEqual("'\\r'", "\r", parser) // Carriage return
507+
assertEqual("'\\t'", "\t", parser) // Tab character
508+
assertEqual("'\\Z'", "\u001A", parser) // ASCII 26 - CTRL + Z (EOF on windows)
509+
510+
// Octals
511+
assertEqual("'\\110\\145\\154\\154\\157\\041'", "Hello!", parser)
512+
513+
// Unicode
514+
assertEqual("'\\u0057\\u006F\\u0072\\u006C\\u0064\\u0020\\u003A\\u0029'", "World :)",
515+
parser)
516516
}
517+
517518
}
518519
}
519520

sql/core/src/main/scala/org/apache/spark/sql/execution/SparkSqlParser.scala

Lines changed: 4 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -39,11 +39,10 @@ import org.apache.spark.sql.types.StructType
3939
/**
4040
* Concrete parser for Spark SQL statements.
4141
*/
42-
class SparkSqlParser extends AbstractSqlParser {
42+
class SparkSqlParser(conf: SQLConf) extends AbstractSqlParser {
43+
val astBuilder = new SparkSqlAstBuilder(conf)
4344

44-
val astBuilder = new SparkSqlAstBuilder
45-
46-
private val substitutor = new VariableSubstitution
45+
private val substitutor = new VariableSubstitution(conf)
4746

4847
protected override def parse[T](command: String)(toResult: SqlBaseParser => T): T = {
4948
super.parse(substitutor.substitute(command))(toResult)
@@ -53,11 +52,9 @@ class SparkSqlParser extends AbstractSqlParser {
5352
/**
5453
* Builder that converts an ANTLR ParseTree into a LogicalPlan/Expression/TableIdentifier.
5554
*/
56-
class SparkSqlAstBuilder extends AstBuilder {
55+
class SparkSqlAstBuilder(conf: SQLConf) extends AstBuilder(conf) {
5756
import org.apache.spark.sql.catalyst.parser.ParserUtils._
5857

59-
private def conf: SQLConf = SQLConf.get
60-
6158
/**
6259
* Create a [[SetCommand]] logical plan.
6360
*

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

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -32,6 +32,7 @@ import org.apache.spark.sql.catalyst.expressions.aggregate._
3232
import org.apache.spark.sql.catalyst.plans.logical.{HintInfo, ResolvedHint}
3333
import org.apache.spark.sql.execution.SparkSqlParser
3434
import org.apache.spark.sql.expressions.UserDefinedFunction
35+
import org.apache.spark.sql.internal.SQLConf
3536
import org.apache.spark.sql.types._
3637
import org.apache.spark.util.Utils
3738

@@ -1275,7 +1276,7 @@ object functions {
12751276
*/
12761277
def expr(expr: String): Column = {
12771278
val parser = SparkSession.getActiveSession.map(_.sessionState.sqlParser).getOrElse {
1278-
new SparkSqlParser
1279+
new SparkSqlParser(new SQLConf)
12791280
}
12801281
Column(parser.parseExpression(expr))
12811282
}

sql/core/src/main/scala/org/apache/spark/sql/internal/BaseSessionStateBuilder.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -114,7 +114,7 @@ abstract class BaseSessionStateBuilder(
114114
* Note: this depends on the `conf` field.
115115
*/
116116
protected lazy val sqlParser: ParserInterface = {
117-
extensions.buildParser(session, new SparkSqlParser)
117+
extensions.buildParser(session, new SparkSqlParser(conf))
118118
}
119119

120120
/**

sql/core/src/main/scala/org/apache/spark/sql/internal/VariableSubstitution.scala

Lines changed: 1 addition & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -25,9 +25,7 @@ import org.apache.spark.internal.config._
2525
*
2626
* Variable substitution is controlled by `SQLConf.variableSubstituteEnabled`.
2727
*/
28-
class VariableSubstitution {
29-
30-
private def conf = SQLConf.get
28+
class VariableSubstitution(conf: SQLConf) {
3129

3230
private val provider = new ConfigProvider {
3331
override def get(key: String): Option[String] = Option(conf.getConfString(key, ""))

sql/core/src/test/scala/org/apache/spark/sql/execution/SparkSqlParserSuite.scala

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -37,7 +37,8 @@ import org.apache.spark.sql.types.{IntegerType, LongType, StringType, StructType
3737
*/
3838
class SparkSqlParserSuite extends AnalysisTest {
3939

40-
private lazy val parser = new SparkSqlParser
40+
val newConf = new SQLConf
41+
private lazy val parser = new SparkSqlParser(newConf)
4142

4243
/**
4344
* Normalizes plans:
@@ -284,7 +285,6 @@ class SparkSqlParserSuite extends AnalysisTest {
284285
}
285286

286287
test("query organization") {
287-
val conf = SQLConf.get
288288
// Test all valid combinations of order by/sort by/distribute by/cluster by/limit/windows
289289
val baseSql = "select * from t"
290290
val basePlan =
@@ -293,20 +293,20 @@ class SparkSqlParserSuite extends AnalysisTest {
293293
assertEqual(s"$baseSql distribute by a, b",
294294
RepartitionByExpression(UnresolvedAttribute("a") :: UnresolvedAttribute("b") :: Nil,
295295
basePlan,
296-
numPartitions = conf.numShufflePartitions))
296+
numPartitions = newConf.numShufflePartitions))
297297
assertEqual(s"$baseSql distribute by a sort by b",
298298
Sort(SortOrder(UnresolvedAttribute("b"), Ascending) :: Nil,
299299
global = false,
300300
RepartitionByExpression(UnresolvedAttribute("a") :: Nil,
301301
basePlan,
302-
numPartitions = conf.numShufflePartitions)))
302+
numPartitions = newConf.numShufflePartitions)))
303303
assertEqual(s"$baseSql cluster by a, b",
304304
Sort(SortOrder(UnresolvedAttribute("a"), Ascending) ::
305305
SortOrder(UnresolvedAttribute("b"), Ascending) :: Nil,
306306
global = false,
307307
RepartitionByExpression(UnresolvedAttribute("a") :: UnresolvedAttribute("b") :: Nil,
308308
basePlan,
309-
numPartitions = conf.numShufflePartitions)))
309+
numPartitions = newConf.numShufflePartitions)))
310310
}
311311

312312
test("pipeline concatenation") {

sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLCommandSuite.scala

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -29,13 +29,13 @@ import org.apache.spark.sql.catalyst.plans.PlanTest
2929
import org.apache.spark.sql.catalyst.plans.logical.Project
3030
import org.apache.spark.sql.execution.SparkSqlParser
3131
import org.apache.spark.sql.execution.datasources.CreateTable
32-
import org.apache.spark.sql.internal.HiveSerDe
32+
import org.apache.spark.sql.internal.{HiveSerDe, SQLConf}
3333
import org.apache.spark.sql.types.{IntegerType, StringType, StructField, StructType}
3434

3535

3636
// TODO: merge this with DDLSuite (SPARK-14441)
3737
class DDLCommandSuite extends PlanTest {
38-
private lazy val parser = new SparkSqlParser
38+
private lazy val parser = new SparkSqlParser(new SQLConf)
3939

4040
private def assertUnsupported(sql: String, containsThesePhrases: Seq[String] = Seq()): Unit = {
4141
val e = intercept[ParseException] {

0 commit comments

Comments
 (0)