diff --git a/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/ExpressionParser.g b/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/ExpressionParser.g index 957bb234e4901cbd57be60e5991c1fa6199753a0..0555a6ba83cbbd0febdef097f4b0b1422f035998 100644 --- a/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/ExpressionParser.g +++ b/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/ExpressionParser.g @@ -167,8 +167,8 @@ intervalLiteral ((intervalConstant KW_HOUR)=> hour=intervalConstant KW_HOUR)? ((intervalConstant KW_MINUTE)=> minute=intervalConstant KW_MINUTE)? ((intervalConstant KW_SECOND)=> second=intervalConstant KW_SECOND)? - (millisecond=intervalConstant KW_MILLISECOND)? - (microsecond=intervalConstant KW_MICROSECOND)? + ((intervalConstant KW_MILLISECOND)=> millisecond=intervalConstant KW_MILLISECOND)? + ((intervalConstant KW_MICROSECOND)=> microsecond=intervalConstant KW_MICROSECOND)? -> ^(TOK_INTERVAL ^(TOK_INTERVAL_YEAR_LITERAL $year?) ^(TOK_INTERVAL_MONTH_LITERAL $month?) @@ -505,10 +505,8 @@ identifier functionIdentifier @init { gParent.pushMsg("function identifier", state); } @after { gParent.popMsg(state); } - : db=identifier DOT fn=identifier - -> Identifier[$db.text + "." + $fn.text] - | - identifier + : + identifier (DOT identifier)? -> identifier+ ; principalIdentifier @@ -553,6 +551,8 @@ nonReserved | KW_SNAPSHOT | KW_AUTOCOMMIT | KW_ANTI + | KW_WEEK | KW_MILLISECOND | KW_MICROSECOND + | KW_CLEAR | KW_LAZY | KW_CACHE | KW_UNCACHE | KW_DFS ; //The following SQL2011 reserved keywords are used as cast function name only, but not as identifiers. diff --git a/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/SparkSqlLexer.g b/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/SparkSqlLexer.g index e4ffc634e8bf456ba1bd2fc08894814f7f3a2750..4374cd7ef7200bc740ddc546cf96eae0aaa3685e 100644 --- a/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/SparkSqlLexer.g +++ b/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/SparkSqlLexer.g @@ -327,6 +327,11 @@ KW_AUTOCOMMIT: 'AUTOCOMMIT'; KW_WEEK: 'WEEK'|'WEEKS'; KW_MILLISECOND: 'MILLISECOND'|'MILLISECONDS'; KW_MICROSECOND: 'MICROSECOND'|'MICROSECONDS'; +KW_CLEAR: 'CLEAR'; +KW_LAZY: 'LAZY'; +KW_CACHE: 'CACHE'; +KW_UNCACHE: 'UNCACHE'; +KW_DFS: 'DFS'; // Operators // NOTE: if you add a new function/operator, add it to sysFuncNames so that describe function _FUNC_ will work. diff --git a/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/SparkSqlParser.g b/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/SparkSqlParser.g index c146ca591488464c7c90595922c8c4c53e8d1bf8..35bef00351d72119ea36cc342d26f7cf06dc8e2d 100644 --- a/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/SparkSqlParser.g +++ b/sql/catalyst/src/main/antlr3/org/apache/spark/sql/catalyst/parser/SparkSqlParser.g @@ -371,6 +371,13 @@ TOK_TXN_READ_WRITE; TOK_COMMIT; TOK_ROLLBACK; TOK_SET_AUTOCOMMIT; +TOK_CACHETABLE; +TOK_UNCACHETABLE; +TOK_CLEARCACHE; +TOK_SETCONFIG; +TOK_DFS; +TOK_ADDFILE; +TOK_ADDJAR; } @@ -515,6 +522,11 @@ import java.util.HashMap; xlateMap.put("KW_WEEK", "WEEK"); xlateMap.put("KW_MILLISECOND", "MILLISECOND"); xlateMap.put("KW_MICROSECOND", "MICROSECOND"); + xlateMap.put("KW_CLEAR", "CLEAR"); + xlateMap.put("KW_LAZY", "LAZY"); + xlateMap.put("KW_CACHE", "CACHE"); + xlateMap.put("KW_UNCACHE", "UNCACHE"); + xlateMap.put("KW_DFS", "DFS"); // Operators xlateMap.put("DOT", "."); @@ -687,8 +699,12 @@ catch (RecognitionException e) { // starting rule statement - : explainStatement EOF - | execStatement EOF + : explainStatement EOF + | execStatement EOF + | KW_ADD KW_JAR -> ^(TOK_ADDJAR) + | KW_ADD KW_FILE -> ^(TOK_ADDFILE) + | KW_DFS -> ^(TOK_DFS) + | (KW_SET)=> KW_SET -> ^(TOK_SETCONFIG) ; explainStatement @@ -717,6 +733,7 @@ execStatement | deleteStatement | updateStatement | sqlTransactionStatement + | cacheStatement ; loadStatement @@ -1390,7 +1407,7 @@ showStatement @init { pushMsg("show statement", state); } @after { popMsg(state); } : KW_SHOW (KW_DATABASES|KW_SCHEMAS) (KW_LIKE showStmtIdentifier)? -> ^(TOK_SHOWDATABASES showStmtIdentifier?) - | KW_SHOW KW_TABLES ((KW_FROM|KW_IN) db_name=identifier)? (KW_LIKE showStmtIdentifier|showStmtIdentifier)? -> ^(TOK_SHOWTABLES (TOK_FROM $db_name)? showStmtIdentifier?) + | KW_SHOW KW_TABLES ((KW_FROM|KW_IN) db_name=identifier)? (KW_LIKE showStmtIdentifier|showStmtIdentifier)? -> ^(TOK_SHOWTABLES ^(TOK_FROM $db_name)? showStmtIdentifier?) | KW_SHOW KW_COLUMNS (KW_FROM|KW_IN) tableName ((KW_FROM|KW_IN) db_name=identifier)? -> ^(TOK_SHOWCOLUMNS tableName $db_name?) | KW_SHOW KW_FUNCTIONS (KW_LIKE showFunctionIdentifier|showFunctionIdentifier)? -> ^(TOK_SHOWFUNCTIONS KW_LIKE? showFunctionIdentifier?) @@ -2438,12 +2455,11 @@ BEGIN user defined transaction boundaries; follows SQL 2003 standard exactly exc sqlTransactionStatement @init { pushMsg("transaction statement", state); } @after { popMsg(state); } - : - startTransactionStatement - | commitStatement - | rollbackStatement - | setAutoCommitStatement - ; + : startTransactionStatement + | commitStatement + | rollbackStatement + | setAutoCommitStatement + ; startTransactionStatement : @@ -2489,3 +2505,31 @@ setAutoCommitStatement /* END user defined transaction boundaries */ + +/* +Table Caching statements. + */ +cacheStatement +@init { pushMsg("cache statement", state); } +@after { popMsg(state); } + : + cacheTableStatement + | uncacheTableStatement + | clearCacheStatement + ; + +cacheTableStatement + : + KW_CACHE (lazy=KW_LAZY)? KW_TABLE identifier (KW_AS selectStatementWithCTE)? -> ^(TOK_CACHETABLE identifier $lazy? selectStatementWithCTE?) + ; + +uncacheTableStatement + : + KW_UNCACHE KW_TABLE identifier -> ^(TOK_UNCACHETABLE identifier) + ; + +clearCacheStatement + : + KW_CLEAR KW_CACHE -> ^(TOK_CLEARCACHE) + ; + diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/CatalystQl.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/CatalystQl.scala index f531d59a75cf8fd1af53c758140b1a22adf6f4da..536c292ab7f34581748e23efac91042bfbe9e89e 100644 --- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/CatalystQl.scala +++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/CatalystQl.scala @@ -210,6 +210,28 @@ https://cwiki.apache.org/confluence/display/Hive/Enhanced+Aggregation%2C+Cube%2C } protected def nodeToPlan(node: ASTNode): LogicalPlan = node match { + case Token("TOK_SHOWFUNCTIONS", args) => + // Skip LIKE. + val pattern = args match { + case like :: nodes if like.text.toUpperCase == "LIKE" => nodes + case nodes => nodes + } + + // Extract Database and Function name + pattern match { + case Nil => + ShowFunctions(None, None) + case Token(name, Nil) :: Nil => + ShowFunctions(None, Some(unquoteString(name))) + case Token(db, Nil) :: Token(name, Nil) :: Nil => + ShowFunctions(Some(unquoteString(db)), Some(unquoteString(name))) + case _ => + noParseRule("SHOW FUNCTIONS", node) + } + + case Token("TOK_DESCFUNCTION", Token(functionName, Nil) :: isExtended) => + DescribeFunction(functionName, isExtended.nonEmpty) + case Token("TOK_QUERY", queryArgs @ Token("TOK_CTE" | "TOK_FROM" | "TOK_INSERT", _) :: _) => val (fromClause: Option[ASTNode], insertClauses, cteRelations) = queryArgs match { diff --git a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/parser/ASTNode.scala b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/parser/ASTNode.scala index ec5e71042d4beb1603399df0609991e7a124eb98..ec9812414e19f8abe1b14e2a74d877b11a2fe04e 100644 --- a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/parser/ASTNode.scala +++ b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/parser/ASTNode.scala @@ -27,10 +27,10 @@ case class ASTNode( children: List[ASTNode], stream: TokenRewriteStream) extends TreeNode[ASTNode] { /** Cache the number of children. */ - val numChildren = children.size + val numChildren: Int = children.size /** tuple used in pattern matching. */ - val pattern = Some((token.getText, children)) + val pattern: Some[(String, List[ASTNode])] = Some((token.getText, children)) /** Line in which the ASTNode starts. */ lazy val line: Int = { @@ -55,10 +55,16 @@ case class ASTNode( } /** Origin of the ASTNode. */ - override val origin = Origin(Some(line), Some(positionInLine)) + override val origin: Origin = Origin(Some(line), Some(positionInLine)) /** Source text. */ - lazy val source = stream.toString(startIndex, stopIndex) + lazy val source: String = stream.toString(startIndex, stopIndex) + + /** Get the source text that remains after this token. */ + lazy val remainder: String = { + stream.fill() + stream.toString(stopIndex + 1, stream.size() - 1).trim() + } def text: String = token.getText diff --git a/sql/core/src/main/scala/org/apache/spark/sql/SQLContext.scala b/sql/core/src/main/scala/org/apache/spark/sql/SQLContext.scala index b774da33aebe4bf8c9db03fbe5ab05ac13f32339..be28df3a51557a42495a12b57c91cc74ba468654 100644 --- a/sql/core/src/main/scala/org/apache/spark/sql/SQLContext.scala +++ b/sql/core/src/main/scala/org/apache/spark/sql/SQLContext.scala @@ -204,7 +204,7 @@ class SQLContext private[sql]( protected[sql] lazy val optimizer: Optimizer = new SparkOptimizer(this) @transient - protected[sql] val sqlParser: ParserInterface = new SparkSQLParser(new SparkQl(conf)) + protected[sql] val sqlParser: ParserInterface = new SparkQl(conf) @transient protected[sql] val ddlParser: DDLParser = new DDLParser(sqlParser) diff --git a/sql/core/src/main/scala/org/apache/spark/sql/execution/SparkQl.scala b/sql/core/src/main/scala/org/apache/spark/sql/execution/SparkQl.scala index f3e89ef4a71f5a9e5833e360e88cd5239edabbe9..f6055306b6c97f6cec07562b6f04bdde40d05fcf 100644 --- a/sql/core/src/main/scala/org/apache/spark/sql/execution/SparkQl.scala +++ b/sql/core/src/main/scala/org/apache/spark/sql/execution/SparkQl.scala @@ -20,6 +20,7 @@ import org.apache.spark.sql.catalyst.{CatalystQl, TableIdentifier} import org.apache.spark.sql.catalyst.analysis.UnresolvedRelation import org.apache.spark.sql.catalyst.parser.{ASTNode, ParserConf, SimpleParserConf} import org.apache.spark.sql.catalyst.plans.logical.{LogicalPlan, OneRowRelation} +import org.apache.spark.sql.catalyst.plans.logical private[sql] class SparkQl(conf: ParserConf = SimpleParserConf()) extends CatalystQl(conf) { /** Check if a command should not be explained. */ @@ -27,6 +28,18 @@ private[sql] class SparkQl(conf: ParserConf = SimpleParserConf()) extends Cataly protected override def nodeToPlan(node: ASTNode): LogicalPlan = { node match { + case Token("TOK_SETCONFIG", Nil) => + val keyValueSeparatorIndex = node.remainder.indexOf('=') + if (keyValueSeparatorIndex >= 0) { + val key = node.remainder.substring(0, keyValueSeparatorIndex).trim + val value = node.remainder.substring(keyValueSeparatorIndex + 1).trim + SetCommand(Some(key -> Option(value))) + } else if (node.remainder.nonEmpty) { + SetCommand(Some(node.remainder -> None)) + } else { + SetCommand(None) + } + // Just fake explain for any of the native commands. case Token("TOK_EXPLAIN", explainArgs) if isNoExplainCommand(explainArgs.head.text) => ExplainCommand(OneRowRelation) @@ -75,6 +88,24 @@ private[sql] class SparkQl(conf: ParserConf = SimpleParserConf()) extends Cataly } } + case Token("TOK_CACHETABLE", Token(tableName, Nil) :: args) => + val Seq(lzy, selectAst) = getClauses(Seq("LAZY", "TOK_QUERY"), args) + CacheTableCommand(tableName, selectAst.map(nodeToPlan), lzy.isDefined) + + case Token("TOK_UNCACHETABLE", Token(tableName, Nil) :: Nil) => + UncacheTableCommand(tableName) + + case Token("TOK_CLEARCACHE", Nil) => + ClearCacheCommand + + case Token("TOK_SHOWTABLES", args) => + val databaseName = args match { + case Nil => None + case Token("TOK_FROM", Token(dbName, Nil) :: Nil) :: Nil => Option(dbName) + case _ => noParseRule("SHOW TABLES", node) + } + ShowTablesCommand(databaseName) + case _ => super.nodeToPlan(node) } diff --git a/sql/core/src/main/scala/org/apache/spark/sql/execution/SparkSQLParser.scala b/sql/core/src/main/scala/org/apache/spark/sql/execution/SparkSQLParser.scala deleted file mode 100644 index d2d827156372653ce5a14bcbc488115604cbbd40..0000000000000000000000000000000000000000 --- a/sql/core/src/main/scala/org/apache/spark/sql/execution/SparkSQLParser.scala +++ /dev/null @@ -1,124 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one or more - * contributor license agreements. See the NOTICE file distributed with - * this work for additional information regarding copyright ownership. - * The ASF licenses this file to You under the Apache License, Version 2.0 - * (the "License"); you may not use this file except in compliance with - * the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, software - * distributed under the License is distributed on an "AS IS" BASIS, - * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. - * See the License for the specific language governing permissions and - * limitations under the License. - */ - -package org.apache.spark.sql.execution - -import scala.util.parsing.combinator.RegexParsers - -import org.apache.spark.sql.catalyst.{AbstractSparkSQLParser, ParserInterface, TableIdentifier} -import org.apache.spark.sql.catalyst.expressions.{Attribute, AttributeReference, Expression} -import org.apache.spark.sql.catalyst.plans.logical -import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan -import org.apache.spark.sql.types.StringType - -/** - * The top level Spark SQL parser. This parser recognizes syntaxes that are available for all SQL - * dialects supported by Spark SQL, and delegates all the other syntaxes to the `fallback` parser. - * - * @param fallback A function that returns the next parser in the chain. This is a call-by-name - * parameter because this allows us to return a different dialect if we - * have to. - */ -class SparkSQLParser(fallback: => ParserInterface) extends AbstractSparkSQLParser { - - override def parseExpression(sql: String): Expression = fallback.parseExpression(sql) - - override def parseTableIdentifier(sql: String): TableIdentifier = - fallback.parseTableIdentifier(sql) - - // A parser for the key-value part of the "SET [key = [value ]]" syntax - private object SetCommandParser extends RegexParsers { - private val key: Parser[String] = "(?m)[^=]+".r - - private val value: Parser[String] = "(?m).*$".r - - private val output: Seq[Attribute] = Seq(AttributeReference("", StringType, nullable = false)()) - - private val pair: Parser[LogicalPlan] = - (key ~ ("=".r ~> value).?).? ^^ { - case None => SetCommand(None) - case Some(k ~ v) => SetCommand(Some(k.trim -> v.map(_.trim))) - } - - def apply(input: String): LogicalPlan = parseAll(pair, input) match { - case Success(plan, _) => plan - case x => sys.error(x.toString) - } - } - - protected val AS = Keyword("AS") - protected val CACHE = Keyword("CACHE") - protected val CLEAR = Keyword("CLEAR") - protected val DESCRIBE = Keyword("DESCRIBE") - protected val EXTENDED = Keyword("EXTENDED") - protected val FUNCTION = Keyword("FUNCTION") - protected val FUNCTIONS = Keyword("FUNCTIONS") - protected val IN = Keyword("IN") - protected val LAZY = Keyword("LAZY") - protected val SET = Keyword("SET") - protected val SHOW = Keyword("SHOW") - protected val TABLE = Keyword("TABLE") - protected val TABLES = Keyword("TABLES") - protected val UNCACHE = Keyword("UNCACHE") - - override protected lazy val start: Parser[LogicalPlan] = - cache | uncache | set | show | desc | others - - private lazy val cache: Parser[LogicalPlan] = - CACHE ~> LAZY.? ~ (TABLE ~> ident) ~ (AS ~> restInput).? ^^ { - case isLazy ~ tableName ~ plan => - CacheTableCommand(tableName, plan.map(fallback.parsePlan), isLazy.isDefined) - } - - private lazy val uncache: Parser[LogicalPlan] = - ( UNCACHE ~ TABLE ~> ident ^^ { - case tableName => UncacheTableCommand(tableName) - } - | CLEAR ~ CACHE ^^^ ClearCacheCommand - ) - - private lazy val set: Parser[LogicalPlan] = - SET ~> restInput ^^ { - case input => SetCommandParser(input) - } - - // It can be the following patterns: - // SHOW FUNCTIONS; - // SHOW FUNCTIONS mydb.func1; - // SHOW FUNCTIONS func1; - // SHOW FUNCTIONS `mydb.a`.`func1.aa`; - private lazy val show: Parser[LogicalPlan] = - ( SHOW ~> TABLES ~ (IN ~> ident).? ^^ { - case _ ~ dbName => ShowTablesCommand(dbName) - } - | SHOW ~ FUNCTIONS ~> ((ident <~ ".").? ~ (ident | stringLit)).? ^^ { - case Some(f) => logical.ShowFunctions(f._1, Some(f._2)) - case None => logical.ShowFunctions(None, None) - } - ) - - private lazy val desc: Parser[LogicalPlan] = - DESCRIBE ~ FUNCTION ~> EXTENDED.? ~ (ident | stringLit) ^^ { - case isExtended ~ functionName => logical.DescribeFunction(functionName, isExtended.isDefined) - } - - private lazy val others: Parser[LogicalPlan] = - wholeInput ^^ { - case input => fallback.parsePlan(input) - } - -} diff --git a/sql/core/src/test/scala/org/apache/spark/sql/SQLQuerySuite.scala b/sql/core/src/test/scala/org/apache/spark/sql/SQLQuerySuite.scala index 10ccd4b8f60db56f7f642a1d2d511f9ea12d35f2..989cb2942918e0f1375286830c3b9b6f665a9455 100644 --- a/sql/core/src/test/scala/org/apache/spark/sql/SQLQuerySuite.scala +++ b/sql/core/src/test/scala/org/apache/spark/sql/SQLQuerySuite.scala @@ -56,8 +56,14 @@ class SQLQuerySuite extends QueryTest with SharedSQLContext { } test("show functions") { - checkAnswer(sql("SHOW functions"), - FunctionRegistry.builtin.listFunction().sorted.map(Row(_))) + def getFunctions(pattern: String): Seq[Row] = { + val regex = java.util.regex.Pattern.compile(pattern) + sqlContext.functionRegistry.listFunction().filter(regex.matcher(_).matches()).map(Row(_)) + } + checkAnswer(sql("SHOW functions"), getFunctions(".*")) + Seq("^c.*", ".*e$", "log.*", ".*date.*").foreach { pattern => + checkAnswer(sql(s"SHOW FUNCTIONS '$pattern'"), getFunctions(pattern)) + } } test("describe functions") { diff --git a/sql/hive/src/main/scala/org/apache/spark/sql/hive/ExtendedHiveQlParser.scala b/sql/hive/src/main/scala/org/apache/spark/sql/hive/ExtendedHiveQlParser.scala deleted file mode 100644 index 313ba18f6aef0f22e1561edfab30a9ad55449064..0000000000000000000000000000000000000000 --- a/sql/hive/src/main/scala/org/apache/spark/sql/hive/ExtendedHiveQlParser.scala +++ /dev/null @@ -1,70 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one or more - * contributor license agreements. See the NOTICE file distributed with - * this work for additional information regarding copyright ownership. - * The ASF licenses this file to You under the Apache License, Version 2.0 - * (the "License"); you may not use this file except in compliance with - * the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, software - * distributed under the License is distributed on an "AS IS" BASIS, - * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. - * See the License for the specific language governing permissions and - * limitations under the License. - */ - -package org.apache.spark.sql.hive - -import scala.language.implicitConversions - -import org.apache.spark.sql.catalyst.{AbstractSparkSQLParser, TableIdentifier} -import org.apache.spark.sql.catalyst.expressions.Expression -import org.apache.spark.sql.catalyst.plans.logical._ -import org.apache.spark.sql.hive.execution.{AddFile, AddJar, HiveNativeCommand} - -/** - * A parser that recognizes all HiveQL constructs together with Spark SQL specific extensions. - */ -private[hive] class ExtendedHiveQlParser(sqlContext: HiveContext) extends AbstractSparkSQLParser { - - val parser = new HiveQl(sqlContext.conf) - - override def parseExpression(sql: String): Expression = parser.parseExpression(sql) - - override def parseTableIdentifier(sql: String): TableIdentifier = - parser.parseTableIdentifier(sql) - - // Keyword is a convention with AbstractSparkSQLParser, which will scan all of the `Keyword` - // properties via reflection the class in runtime for constructing the SqlLexical object - protected val ADD = Keyword("ADD") - protected val DFS = Keyword("DFS") - protected val FILE = Keyword("FILE") - protected val JAR = Keyword("JAR") - - protected lazy val start: Parser[LogicalPlan] = dfs | addJar | addFile | hiveQl - - protected lazy val hiveQl: Parser[LogicalPlan] = - restInput ^^ { - case statement => - sqlContext.executionHive.withHiveState { - parser.parsePlan(statement.trim) - } - } - - protected lazy val dfs: Parser[LogicalPlan] = - DFS ~> wholeInput ^^ { - case command => HiveNativeCommand(command.trim) - } - - private lazy val addFile: Parser[LogicalPlan] = - ADD ~ FILE ~> restInput ^^ { - case input => AddFile(input.trim) - } - - private lazy val addJar: Parser[LogicalPlan] = - ADD ~ JAR ~> restInput ^^ { - case input => AddJar(input.trim) - } -} diff --git a/sql/hive/src/main/scala/org/apache/spark/sql/hive/HiveContext.scala b/sql/hive/src/main/scala/org/apache/spark/sql/hive/HiveContext.scala index eaca3c9269bb7a25ffae6a33a568a31d529ce484..1797ea54f2501f9493ae61d5989c5b7f3e059556 100644 --- a/sql/hive/src/main/scala/org/apache/spark/sql/hive/HiveContext.scala +++ b/sql/hive/src/main/scala/org/apache/spark/sql/hive/HiveContext.scala @@ -316,7 +316,9 @@ class HiveContext private[hive]( } protected[sql] override def parseSql(sql: String): LogicalPlan = { - super.parseSql(substitutor.substitute(hiveconf, sql)) + executionHive.withHiveState { + super.parseSql(substitutor.substitute(hiveconf, sql)) + } } override protected[sql] def executePlan(plan: LogicalPlan): this.QueryExecution = @@ -546,9 +548,7 @@ class HiveContext private[hive]( } @transient - protected[sql] override val sqlParser: ParserInterface = { - new SparkSQLParser(new ExtendedHiveQlParser(this)) - } + protected[sql] override val sqlParser: ParserInterface = new HiveQl(conf) @transient private val hivePlanner = new SparkPlanner(this) with HiveStrategies { diff --git a/sql/hive/src/main/scala/org/apache/spark/sql/hive/HiveQl.scala b/sql/hive/src/main/scala/org/apache/spark/sql/hive/HiveQl.scala index 46246f8191db10aed0b1e28699197b8523362020..22841ed2116d1ddcd1cb7463b639f5210ffe07e3 100644 --- a/sql/hive/src/main/scala/org/apache/spark/sql/hive/HiveQl.scala +++ b/sql/hive/src/main/scala/org/apache/spark/sql/hive/HiveQl.scala @@ -35,11 +35,12 @@ import org.apache.spark.sql.catalyst.TableIdentifier import org.apache.spark.sql.catalyst.expressions._ import org.apache.spark.sql.catalyst.parser._ import org.apache.spark.sql.catalyst.parser.ParseUtils._ +import org.apache.spark.sql.catalyst.plans._ import org.apache.spark.sql.catalyst.plans.logical._ import org.apache.spark.sql.execution.SparkQl import org.apache.spark.sql.hive.HiveShim.HiveFunctionWrapper import org.apache.spark.sql.hive.client._ -import org.apache.spark.sql.hive.execution.{AnalyzeTable, DropTable, HiveNativeCommand, HiveScriptIOSchema} +import org.apache.spark.sql.hive.execution._ import org.apache.spark.sql.types._ import org.apache.spark.sql.AnalysisException @@ -113,7 +114,6 @@ private[hive] class HiveQl(conf: ParserConf) extends SparkQl(conf) with Logging "TOK_CREATEROLE", "TOK_DESCDATABASE", - "TOK_DESCFUNCTION", "TOK_DROPDATABASE", "TOK_DROPFUNCTION", @@ -151,7 +151,6 @@ private[hive] class HiveQl(conf: ParserConf) extends SparkQl(conf) with Logging "TOK_SHOW_TRANSACTIONS", "TOK_SHOWCOLUMNS", "TOK_SHOWDATABASES", - "TOK_SHOWFUNCTIONS", "TOK_SHOWINDEXES", "TOK_SHOWLOCKS", "TOK_SHOWPARTITIONS", @@ -244,6 +243,15 @@ private[hive] class HiveQl(conf: ParserConf) extends SparkQl(conf) with Logging protected override def nodeToPlan(node: ASTNode): LogicalPlan = { node match { + case Token("TOK_DFS", Nil) => + HiveNativeCommand(node.source + " " + node.remainder) + + case Token("TOK_ADDFILE", Nil) => + AddFile(node.remainder) + + case Token("TOK_ADDJAR", Nil) => + AddJar(node.remainder) + // Special drop table that also uncaches. case Token("TOK_DROPTABLE", Token("TOK_TABNAME", tableNameParts) :: ifExists) => val tableName = tableNameParts.map { case Token(p, Nil) => p }.mkString(".") @@ -558,7 +566,7 @@ private[hive] class HiveQl(conf: ParserConf) extends SparkQl(conf) with Logging protected override def nodeToTransformation( node: ASTNode, - child: LogicalPlan): Option[ScriptTransformation] = node match { + child: LogicalPlan): Option[logical.ScriptTransformation] = node match { case Token("TOK_SELEXPR", Token("TOK_TRANSFORM", Token("TOK_EXPLIST", inputExprs) :: @@ -651,7 +659,7 @@ private[hive] class HiveQl(conf: ParserConf) extends SparkQl(conf) with Logging schemaLess) Some( - ScriptTransformation( + logical.ScriptTransformation( inputExprs.map(nodeToExpr), unescapedScript, output, diff --git a/sql/hive/src/test/resources/golden/show_functions-1-4a6f611305f58bdbafb2fd89ec62d797 b/sql/hive/src/test/resources/golden/show_functions-1-4a6f611305f58bdbafb2fd89ec62d797 index 175795534fff516ec8fc4ca407c75b7d076438ba..f400819b67c26ada075eebe6a2db3211b18cee13 100644 --- a/sql/hive/src/test/resources/golden/show_functions-1-4a6f611305f58bdbafb2fd89ec62d797 +++ b/sql/hive/src/test/resources/golden/show_functions-1-4a6f611305f58bdbafb2fd89ec62d797 @@ -1,4 +1,5 @@ case +cbrt ceil ceiling coalesce @@ -17,3 +18,6 @@ covar_samp create_union cume_dist current_database +current_date +current_timestamp +current_user diff --git a/sql/hive/src/test/resources/golden/show_functions-2-97cbada21ad9efda7ce9de5891deca7c b/sql/hive/src/test/resources/golden/show_functions-2-97cbada21ad9efda7ce9de5891deca7c index 3c25d656bda1cb5b180e697d2ec6cbec42426b18..19458fc86e439f8d579d28a29573390b1392ce90 100644 --- a/sql/hive/src/test/resources/golden/show_functions-2-97cbada21ad9efda7ce9de5891deca7c +++ b/sql/hive/src/test/resources/golden/show_functions-2-97cbada21ad9efda7ce9de5891deca7c @@ -2,6 +2,7 @@ assert_true case coalesce current_database +current_date decode e encode diff --git a/sql/hive/src/test/resources/golden/show_functions-4-4deaa213aff83575bbaf859f79bfdd48 b/sql/hive/src/test/resources/golden/show_functions-4-4deaa213aff83575bbaf859f79bfdd48 index cd2e58d04a4ef0c859a397661fedbbcdb74d6056..1d05f843a7e0f5a5d8c516d2ef8c474a5e3b3ca2 100644 --- a/sql/hive/src/test/resources/golden/show_functions-4-4deaa213aff83575bbaf859f79bfdd48 +++ b/sql/hive/src/test/resources/golden/show_functions-4-4deaa213aff83575bbaf859f79bfdd48 @@ -1,4 +1,6 @@ +current_date date_add +date_format date_sub datediff to_date diff --git a/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/SQLQuerySuite.scala b/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/SQLQuerySuite.scala index 9e53d8a81e75370dfa277c4b9b95fc1e2dd38c02..0d62d799c8dce8e8123c46b1fa8ef59c2f8d607f 100644 --- a/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/SQLQuerySuite.scala +++ b/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/SQLQuerySuite.scala @@ -28,7 +28,7 @@ import org.apache.spark.sql.catalyst.parser.ParserConf import org.apache.spark.sql.execution.SparkQl import org.apache.spark.sql.execution.datasources.LogicalRelation import org.apache.spark.sql.execution.datasources.parquet.ParquetRelation -import org.apache.spark.sql.hive.{ExtendedHiveQlParser, HiveContext, HiveQl, MetastoreRelation} +import org.apache.spark.sql.hive.{HiveContext, HiveQl, MetastoreRelation} import org.apache.spark.sql.hive.test.TestHiveSingleton import org.apache.spark.sql.test.SQLTestUtils import org.apache.spark.sql.types._