Skip to content

Commit

Permalink
ddl: Function related commands
Browse files Browse the repository at this point in the history
  • Loading branch information
bomeng committed Mar 26, 2016
1 parent a91784f commit 033b1ac
Show file tree
Hide file tree
Showing 3 changed files with 28 additions and 10 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -21,8 +21,9 @@ import java.util.NoSuchElementException

import org.apache.spark.internal.Logging
import org.apache.spark.rdd.RDD
import org.apache.spark.sql.catalyst.catalog.CatalogFunction
import org.apache.spark.sql.{Dataset, Row, SQLContext}
import org.apache.spark.sql.catalyst.{CatalystTypeConverters, InternalRow, TableIdentifier}
import org.apache.spark.sql.catalyst.{CatalystTypeConverters, FunctionIdentifier, InternalRow, TableIdentifier}
import org.apache.spark.sql.catalyst.errors.TreeNodeException
import org.apache.spark.sql.catalyst.expressions.{Attribute, AttributeReference}
import org.apache.spark.sql.catalyst.plans.logical
Expand Down Expand Up @@ -434,3 +435,27 @@ case class SetDatabaseCommand(databaseName: String) extends RunnableCommand {

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

/**
* A command for users to create a function.
* The syntax of using this command in SQL is
* {{{
* CREATE TEMPORARY FUNCTION function_name AS class_name;
* CREATE FUNCTION [db_name.]function_name AS class_name
* [USING JAR|FILE|ARCHIVE 'file_uri' [, JAR|FILE|ARCHIVE 'file_uri'] ];
* }}}
*/
case class CreateFunction(
functionName: String,
alias: String,
resources: Seq[(String, String)],
isTemp: Boolean)(sql: String) extends RunnableCommand {
override def run(sqlContext: SQLContext): Seq[Row] = {
val catalog = sqlContext.sessionState.catalog
val functionIdentifier = FunctionIdentifier(functionName, Some(catalog.getCurrentDatabase))
catalog.createFunction(CatalogFunction(functionIdentifier, alias))
Seq.empty[Row]
}

override val output: Seq[Attribute] = Seq.empty
}
Original file line number Diff line number Diff line change
Expand Up @@ -52,13 +52,6 @@ case class CreateDatabase(
props: Map[String, String])(sql: String)
extends NativeDDLCommand(sql) with Logging

case class CreateFunction(
functionName: String,
alias: String,
resources: Seq[(String, String)],
isTemp: Boolean)(sql: String)
extends NativeDDLCommand(sql) with Logging

case class AlterTableRename(
oldName: TableIdentifier,
newName: TableIdentifier)(sql: String)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -47,13 +47,13 @@ class DDLCommandSuite extends PlanTest {
test("create function") {
val sql1 =
"""
|CREATE TEMPORARY FUNCTION helloworld as
|CREATE TEMPORARY FUNCTION helloworld AS
|'com.matthewrathbone.example.SimpleUDFExample' USING JAR '/path/to/jar1',
|JAR '/path/to/jar2'
""".stripMargin
val sql2 =
"""
|CREATE FUNCTION hello.world as
|CREATE FUNCTION hello.world AS
|'com.matthewrathbone.example.SimpleUDFExample' USING ARCHIVE '/path/to/archive',
|FILE '/path/to/file'
""".stripMargin
Expand Down

0 comments on commit 033b1ac

Please sign in to comment.