Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
Show all changes
53 commits
Select commit Hold shift + click to select a range
01e4cdf
Merge remote-tracking branch 'upstream/master'
gatorsmile Nov 13, 2015
6835704
Merge remote-tracking branch 'upstream/master'
gatorsmile Nov 14, 2015
9180687
Merge remote-tracking branch 'upstream/master'
gatorsmile Nov 14, 2015
b38a21e
SPARK-11633
gatorsmile Nov 17, 2015
d2b84af
Merge remote-tracking branch 'upstream/master' into joinMakeCopy
gatorsmile Nov 17, 2015
fda8025
Merge remote-tracking branch 'upstream/master'
gatorspark Nov 17, 2015
ac0dccd
Merge branch 'master' of https://github.com/gatorsmile/spark
gatorspark Nov 17, 2015
6e0018b
Merge remote-tracking branch 'upstream/master'
Nov 20, 2015
0546772
converge
gatorsmile Nov 20, 2015
b37a64f
converge
gatorsmile Nov 20, 2015
c2a872c
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 6, 2016
ab6dbd7
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 6, 2016
4276356
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 6, 2016
2dab708
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 7, 2016
0458770
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 8, 2016
1debdfa
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 9, 2016
763706d
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 14, 2016
4de6ec1
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 18, 2016
9422a4f
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 19, 2016
52bdf48
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 20, 2016
1e95df3
Merge remote-tracking branch 'upstream/master'
gatorsmile Jan 23, 2016
fab24cf
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 1, 2016
8b2e33b
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 5, 2016
2ee1876
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 11, 2016
b9f0090
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 12, 2016
ade6f7e
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 15, 2016
9fd63d2
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 19, 2016
5199d49
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 22, 2016
404214c
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 23, 2016
c001dd9
Merge remote-tracking branch 'upstream/master'
gatorsmile Feb 25, 2016
59daa48
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 5, 2016
41d5f64
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 7, 2016
472a6e3
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 10, 2016
0fba10a
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 12, 2016
cbf73b3
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 21, 2016
c08f561
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 22, 2016
474df88
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 22, 2016
3d9828d
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 24, 2016
602cace
native DDL support for create database
gatorsmile Mar 25, 2016
72d2361
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 26, 2016
e26542e
Merge remote-tracking branch 'upstream/master' into dbDDL
gatorsmile Mar 27, 2016
2794bfa
set the directory for database creation
gatorsmile Mar 27, 2016
65ce660
add a default db extension.
gatorsmile Mar 28, 2016
aba3a95
native support for drop/alter/describe database
gatorsmile Mar 28, 2016
ffe0c7a
add more test cases.
gatorsmile Mar 28, 2016
d7c3648
added an extra line.
gatorsmile Mar 28, 2016
9420f08
added more comments.
gatorsmile Mar 28, 2016
4dab82c
update the comment.
gatorsmile Mar 28, 2016
f4c33e2
update the comment.
gatorsmile Mar 28, 2016
536cf36
address comments.
gatorsmile Mar 29, 2016
16c829e
address comments.
gatorsmile Mar 29, 2016
07afea5
Merge remote-tracking branch 'upstream/master'
gatorsmile Mar 29, 2016
f22ef90
Merge branch 'dbDDL' into dbDDLnew
gatorsmile Mar 29, 2016
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 @@ -17,6 +17,8 @@

package org.apache.spark.sql.catalyst.catalog

import java.io.File

import scala.collection.mutable

import org.apache.spark.sql.AnalysisException
Expand Down Expand Up @@ -114,6 +116,10 @@ class SessionCatalog(
currentDb = db
}

def getDefaultDBPath(db: String): String = {
System.getProperty("java.io.tmpdir") + File.separator + db + ".db"
}

// ----------------------------------------------------------------------------
// Tables
// ----------------------------------------------------------------------------
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -39,7 +39,7 @@ abstract class ExternalCatalog {

protected def requireDbExists(db: String): Unit = {
if (!databaseExists(db)) {
throw new AnalysisException(s"Database $db does not exist")
throw new AnalysisException(s"Database '$db' does not exist")
Copy link
Contributor

Choose a reason for hiding this comment

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

maybe use backtick?

Copy link
Member Author

Choose a reason for hiding this comment

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

Ok. Sure. We need to change all the places.

Copy link
Contributor

Choose a reason for hiding this comment

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

let's change it in a separate patch then. There are a few other places like this

Copy link
Member Author

Choose a reason for hiding this comment

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

Sure, will do it in a separate PR.

}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -97,7 +97,8 @@ private[sql] class SparkQl(conf: ParserConf = SimpleParserConf()) extends Cataly

// CREATE DATABASE [IF NOT EXISTS] database_name [COMMENT database_comment]
// [LOCATION path] [WITH DBPROPERTIES (key1=val1, key2=val2, ...)];
case Token("TOK_CREATEDATABASE", Token(databaseName, Nil) :: args) =>
case Token("TOK_CREATEDATABASE", Token(dbName, Nil) :: args) =>
Copy link
Contributor

Choose a reason for hiding this comment

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

changes in this file aren't really necessary since the file will be deleted anyway, but for now it's OK to keep

val databaseName = cleanIdentifier(dbName)
val Seq(ifNotExists, dbLocation, databaseComment, dbprops) = getClauses(Seq(
"TOK_IFNOTEXISTS",
"TOK_DATABASELOCATION",
Expand Down Expand Up @@ -126,7 +127,7 @@ private[sql] class SparkQl(conf: ParserConf = SimpleParserConf()) extends Cataly
extractProps(propList, "TOK_TABLEPROPERTY")
case _ => parseFailed("Invalid CREATE DATABASE command", node)
}.toMap
CreateDatabase(databaseName, ifNotExists.isDefined, location, comment, props)(node.source)
CreateDatabase(databaseName, ifNotExists.isDefined, location, comment, props)

// DROP DATABASE [IF EXISTS] database_name [RESTRICT|CASCADE];
case Token("TOK_DROPDATABASE", Token(dbName, Nil) :: otherArgs) =>
Expand All @@ -136,15 +137,15 @@ private[sql] class SparkQl(conf: ParserConf = SimpleParserConf()) extends Cataly
// :- database_name
// :- TOK_IFEXISTS
// +- TOK_RESTRICT/TOK_CASCADE
val databaseName = unquoteString(dbName)
val databaseName = cleanIdentifier(dbName)
// The default is RESTRICT
val Seq(ifExists, _, cascade) = getClauses(Seq(
"TOK_IFEXISTS", "TOK_RESTRICT", "TOK_CASCADE"), otherArgs)
DropDatabase(databaseName, ifExists.isDefined, restrict = cascade.isEmpty)(node.source)
DropDatabase(databaseName, ifExists.isDefined, cascade.isDefined)

// ALTER (DATABASE|SCHEMA) database_name SET DBPROPERTIES (property_name=property_value, ...)
case Token("TOK_ALTERDATABASE_PROPERTIES", Token(dbName, Nil) :: args) =>
val databaseName = unquoteString(dbName)
val databaseName = cleanIdentifier(dbName)
val dbprops = getClause("TOK_DATABASEPROPERTIES", args)
val props = dbprops match {
case Token("TOK_DATABASEPROPERTIES", Token("TOK_DBPROPLIST", propList) :: Nil) =>
Expand All @@ -161,13 +162,13 @@ private[sql] class SparkQl(conf: ParserConf = SimpleParserConf()) extends Cataly
extractProps(propList, "TOK_TABLEPROPERTY")
case _ => parseFailed("Invalid ALTER DATABASE command", node)
}
AlterDatabaseProperties(databaseName, props.toMap)(node.source)
AlterDatabaseProperties(databaseName, props.toMap)

// DESCRIBE DATABASE [EXTENDED] db_name
case Token("TOK_DESCDATABASE", Token(dbName, Nil) :: describeArgs) =>
val databaseName = unquoteString(dbName)
val databaseName = cleanIdentifier(dbName)
val extended = getClauseOption("EXTENDED", describeArgs)
DescribeDatabase(databaseName, extended.isDefined)(node.source)
DescribeDatabase(databaseName, extended.isDefined)

// CREATE [TEMPORARY] FUNCTION [db_name.]function_name AS class_name
// [USING JAR|FILE|ARCHIVE 'file_uri' [, JAR|FILE|ARCHIVE 'file_uri'] ];
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -232,8 +232,7 @@ class SparkSqlAstBuilder extends AstBuilder {
ctx.EXISTS != null,
Option(ctx.locationSpec).map(visitLocationSpec),
Option(ctx.comment).map(string),
Option(ctx.tablePropertyList).map(visitTablePropertyList).getOrElse(Map.empty))(
command(ctx))
Option(ctx.tablePropertyList).map(visitTablePropertyList).getOrElse(Map.empty))
}

/**
Expand All @@ -248,8 +247,7 @@ class SparkSqlAstBuilder extends AstBuilder {
ctx: SetDatabasePropertiesContext): LogicalPlan = withOrigin(ctx) {
AlterDatabaseProperties(
ctx.identifier.getText,
visitTablePropertyList(ctx.tablePropertyList))(
command(ctx))
visitTablePropertyList(ctx.tablePropertyList))
}

/**
Expand All @@ -261,7 +259,7 @@ class SparkSqlAstBuilder extends AstBuilder {
* }}}
*/
override def visitDropDatabase(ctx: DropDatabaseContext): LogicalPlan = withOrigin(ctx) {
DropDatabase(ctx.identifier.getText, ctx.EXISTS != null, ctx.CASCADE == null)(command(ctx))
DropDatabase(ctx.identifier.getText, ctx.EXISTS != null, ctx.CASCADE != null)
}

/**
Expand All @@ -273,7 +271,7 @@ class SparkSqlAstBuilder extends AstBuilder {
* }}}
*/
override def visitDescribeDatabase(ctx: DescribeDatabaseContext): LogicalPlan = withOrigin(ctx) {
DescribeDatabase(ctx.identifier.getText, ctx.EXTENDED != null)(command(ctx))
DescribeDatabase(ctx.identifier.getText, ctx.EXTENDED != null)
}

/**
Expand Down
125 changes: 107 additions & 18 deletions sql/core/src/main/scala/org/apache/spark/sql/execution/command/ddl.scala
Original file line number Diff line number Diff line change
Expand Up @@ -19,8 +19,8 @@ package org.apache.spark.sql.execution.command

import org.apache.spark.internal.Logging
import org.apache.spark.sql.{Row, SQLContext}
import org.apache.spark.sql.catalyst.{FunctionIdentifier, TableIdentifier}
import org.apache.spark.sql.catalyst.catalog.CatalogFunction
import org.apache.spark.sql.catalyst.TableIdentifier
import org.apache.spark.sql.catalyst.catalog.CatalogDatabase
import org.apache.spark.sql.catalyst.catalog.ExternalCatalog.TablePartitionSpec
import org.apache.spark.sql.catalyst.expressions.{Attribute, AttributeReference}
import org.apache.spark.sql.execution.datasources.BucketSpec
Expand All @@ -45,46 +45,135 @@ abstract class NativeDDLCommand(val sql: String) extends RunnableCommand {

}

/**
* A command for users to create a new database.
*
* It will issue an error message when the database with the same name already exists,
* unless 'ifNotExists' is true.
* The syntax of using this command in SQL is:
* {{{
* CREATE DATABASE|SCHEMA [IF NOT EXISTS] database_name
* }}}
*/
case class CreateDatabase(
databaseName: String,
ifNotExists: Boolean,
path: Option[String],
comment: Option[String],
props: Map[String, String])(sql: String)
extends NativeDDLCommand(sql) with Logging
props: Map[String, String])
extends RunnableCommand {

override def run(sqlContext: SQLContext): Seq[Row] = {
val catalog = sqlContext.sessionState.catalog
catalog.createDatabase(
CatalogDatabase(
databaseName,
comment.getOrElse(""),
path.getOrElse(catalog.getDefaultDBPath(databaseName)),
props),
ifNotExists)
Seq.empty[Row]
}
Copy link
Contributor

Choose a reason for hiding this comment

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

Do we need to create the underlying dir in this command?

Copy link
Member Author

Choose a reason for hiding this comment

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

True. Only did it in the default DB path. I forgot to do it in the regular case : ( Let me submit a follow-up PR for it. Thanks for catching it!

Copy link
Member Author

Choose a reason for hiding this comment

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

@yhuai I tried it in spark-sql. If the directory is not created, Hive will do it for us. I am wondering if we still should create directory in Spark?

However, this PR has an issue when users specify the location in the Create Database command. The generated path should be path/databaseName.db instead of path. Will fix it soon.

Copy link
Contributor

Choose a reason for hiding this comment

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

Yea. Let's create the directory if it is not created.

Copy link
Member Author

Choose a reason for hiding this comment

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

I see. Will do it in #12081. Thanks!

Copy link
Member Author

Choose a reason for hiding this comment

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

@yhuai I did try it. Actually, the code is done... However, if we create a directory before issuing Hive client API createDatabase, we will get the following error message from Hive:

Error in query: org.apache.hadoop.hive.metastore.api.AlreadyExistsException: Database db3 already exists;

Just feel free to let me know what I should do next. Thanks!


override val output: Seq[Attribute] = Seq.empty
}


/**
* Drop Database: Removes a database from the system.
* A command for users to remove a database from the system.
*
* 'ifExists':
* - true, if database_name does't exist, no action
* - false (default), if database_name does't exist, a warning message will be issued
* 'restric':
* - true (default), the database cannot be dropped if it is not empty. The inclusive
* tables must be dropped at first.
* - false, it is in the Cascade mode. The dependent objects are automatically dropped
* before dropping database.
* 'cascade':
* - true, the dependent objects are automatically dropped before dropping database.
* - false (default), it is in the Restrict mode. The database cannot be dropped if
* it is not empty. The inclusive tables must be dropped at first.
*
* The syntax of using this command in SQL is:
* {{{
* DROP DATABASE [IF EXISTS] database_name [RESTRICT|CASCADE];
* }}}
*/
case class DropDatabase(
databaseName: String,
ifExists: Boolean,
restrict: Boolean)(sql: String)
extends NativeDDLCommand(sql) with Logging
cascade: Boolean)
extends RunnableCommand {

override def run(sqlContext: SQLContext): Seq[Row] = {
sqlContext.sessionState.catalog.dropDatabase(databaseName, ifExists, cascade)
Seq.empty[Row]
}

override val output: Seq[Attribute] = Seq.empty
}

/** ALTER DATABASE: add new (key, value) pairs into DBPROPERTIES */
/**
* A command for users to add new (key, value) pairs into DBPROPERTIES
* If the database does not exist, an error message will be issued to indicate the database
* does not exist.
* The syntax of using this command in SQL is:
* {{{
* ALTER (DATABASE|SCHEMA) database_name SET DBPROPERTIES (property_name=property_value, ...)
* }}}
*/
case class AlterDatabaseProperties(
databaseName: String,
props: Map[String, String])(sql: String)
extends NativeDDLCommand(sql) with Logging
props: Map[String, String])
extends RunnableCommand {

override def run(sqlContext: SQLContext): Seq[Row] = {
val catalog = sqlContext.sessionState.catalog
val db: CatalogDatabase = catalog.getDatabase(databaseName)
catalog.alterDatabase(db.copy(properties = db.properties ++ props))

Seq.empty[Row]
}

override val output: Seq[Attribute] = Seq.empty
}

/**
* DESCRIBE DATABASE: shows the name of the database, its comment (if one has been set), and its
* A command for users to show the name of the database, its comment (if one has been set), and its
* root location on the filesystem. When extended is true, it also shows the database's properties
* If the database does not exist, an error message will be issued to indicate the database
* does not exist.
* The syntax of using this command in SQL is
* {{{
* DESCRIBE DATABASE [EXTENDED] db_name
* }}}
*/
case class DescribeDatabase(
databaseName: String,
extended: Boolean)(sql: String)
extends NativeDDLCommand(sql) with Logging
extended: Boolean)
extends RunnableCommand {

override def run(sqlContext: SQLContext): Seq[Row] = {
val dbMetadata: CatalogDatabase = sqlContext.sessionState.catalog.getDatabase(databaseName)
val result =
Row("Database Name", dbMetadata.name) ::
Copy link
Contributor

Choose a reason for hiding this comment

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

this can probably just be Name. The user ran DESCRIBE DATABASE so it's pretty obvious

Row("Description", dbMetadata.description) ::
Row("Location", dbMetadata.locationUri) :: Nil

if (extended) {
val properties =
if (dbMetadata.properties.isEmpty) {
""
} else {
dbMetadata.properties.toSeq.mkString("(", ", ", ")")
}
result :+ Row("Properties", properties)
} else {
result
}
}

override val output: Seq[Attribute] = {
AttributeReference("database_description_item", StringType, nullable = false)() ::
AttributeReference("database_description_value", StringType, nullable = false)() :: Nil
}
}

case class CreateFunction(
databaseName: Option[String],
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -39,7 +39,7 @@ class DDLCommandSuite extends PlanTest {
ifNotExists = true,
Some("/home/user/db"),
Some("database_comment"),
Map("a" -> "a", "b" -> "b", "c" -> "c"))(sql)
Map("a" -> "a", "b" -> "b", "c" -> "c"))
comparePlans(parsed, expected)
}

Expand All @@ -65,39 +65,27 @@ class DDLCommandSuite extends PlanTest {
val expected1 = DropDatabase(
"database_name",
ifExists = true,
restrict = true)(sql1)
cascade = false)
val expected2 = DropDatabase(
"database_name",
ifExists = true,
restrict = false)(sql2)
cascade = true)
val expected3 = DropDatabase(
"database_name",
ifExists = true,
restrict = true)(sql3)
val expected4 = DropDatabase(
"database_name",
ifExists = true,
restrict = false)(sql4)
val expected5 = DropDatabase(
"database_name",
ifExists = true,
restrict = true)(sql5)
val expected6 = DropDatabase(
"database_name",
ifExists = false,
restrict = true)(sql6)
val expected7 = DropDatabase(
cascade = false)
val expected4 = DropDatabase(
"database_name",
ifExists = false,
restrict = false)(sql7)
cascade = true)

comparePlans(parsed1, expected1)
comparePlans(parsed2, expected2)
comparePlans(parsed3, expected3)
comparePlans(parsed4, expected4)
comparePlans(parsed5, expected5)
comparePlans(parsed6, expected6)
comparePlans(parsed7, expected7)
comparePlans(parsed3, expected1)
comparePlans(parsed4, expected2)
comparePlans(parsed5, expected1)
comparePlans(parsed6, expected3)
comparePlans(parsed7, expected4)
}

test("alter database set dbproperties") {
Expand All @@ -110,10 +98,10 @@ class DDLCommandSuite extends PlanTest {

val expected1 = AlterDatabaseProperties(
"database_name",
Map("a" -> "a", "b" -> "b", "c" -> "c"))(sql1)
Map("a" -> "a", "b" -> "b", "c" -> "c"))
val expected2 = AlterDatabaseProperties(
"database_name",
Map("a" -> "a"))(sql2)
Map("a" -> "a"))

comparePlans(parsed1, expected1)
comparePlans(parsed2, expected2)
Expand All @@ -129,10 +117,10 @@ class DDLCommandSuite extends PlanTest {

val expected1 = DescribeDatabase(
"db_name",
extended = true)(sql1)
extended = true)
val expected2 = DescribeDatabase(
"db_name",
extended = false)(sql2)
extended = false)

comparePlans(parsed1, expected1)
comparePlans(parsed2, expected2)
Expand Down
Loading