Skip to content
Closed
Show file tree
Hide file tree
Changes from 6 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
Original file line number Diff line number Diff line change
Expand Up @@ -17,13 +17,15 @@

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

import java.lang.reflect.InvocationTargetException
import java.net.URI
import java.util.Locale
import java.util.concurrent.Callable
import javax.annotation.concurrent.GuardedBy

import scala.collection.mutable
import scala.util.{Failure, Success, Try}
import scala.util.control.NonFatal

import com.google.common.cache.{Cache, CacheBuilder}
import org.apache.hadoop.conf.Configuration
Expand All @@ -39,7 +41,9 @@ import org.apache.spark.sql.catalyst.parser.{CatalystSqlParser, ParserInterface}
import org.apache.spark.sql.catalyst.plans.logical.{LogicalPlan, SubqueryAlias, View}
import org.apache.spark.sql.catalyst.util.StringUtils
import org.apache.spark.sql.internal.SQLConf
import org.apache.spark.sql.internal.StaticSQLConf.CATALOG_IMPLEMENTATION
import org.apache.spark.sql.types.StructType
import org.apache.spark.util.Utils

object SessionCatalog {
val DEFAULT_DATABASE = "default"
Expand Down Expand Up @@ -126,6 +130,13 @@ class SessionCatalog(
if (conf.caseSensitiveAnalysis) name else name.toLowerCase(Locale.ROOT)
}

/**
* Checks whether the Hive metastore is being used
*/
private def isUsingHiveMetastore: Boolean = {
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 still need this?

conf.getConf(CATALOG_IMPLEMENTATION).toLowerCase(Locale.ROOT) == "hive"
}

private val tableRelationCache: Cache[QualifiedTableName, LogicalPlan] = {
val cacheSize = conf.tableRelationCacheSize
CacheBuilder.newBuilder().maximumSize(cacheSize).build[QualifiedTableName, LogicalPlan]()
Expand Down Expand Up @@ -1090,14 +1101,50 @@ class SessionCatalog(
// | Methods that interact with temporary and metastore functions |
// ----------------------------------------------------------------

/**
* Constructs a [[FunctionBuilder]] based on the provided class that represents a function.
*/
protected def makeFunctionBuilder(name: String, functionClassName: String): FunctionBuilder = {
Copy link
Contributor

@cloud-fan cloud-fan Aug 14, 2017

Choose a reason for hiding this comment

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

this will be overwritten by HiveSessionCatalog, does it mean we can not register spark UDAF if hive support is enabled?

Copy link
Member Author

Choose a reason for hiding this comment

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

The changes here are for HiveSessionCatalog. Also, we have a test case in HiveUDAFSuite.scala to verify it.

val clazz = Utils.classForName(functionClassName)
(children: Seq[Expression]) => {
try {
makeFunctionExpression(name, Utils.classForName(functionClassName), children).getOrElse {
Copy link
Contributor

Choose a reason for hiding this comment

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

Utils.classForName(functionClassName) -> clazz

val extraMsg =
if (!isUsingHiveMetastore) "Use sparkSession.udf.register(...) instead." else ""
throw new AnalysisException(
s"No handler for UDF/UDAF/UDTF '${clazz.getCanonicalName}'. $extraMsg")
}
} catch {
case ae: AnalysisException =>
throw ae
case NonFatal(e) =>
val analysisException =
new AnalysisException(s"No handler for UDF/UDAF/UDTF '${clazz.getCanonicalName}': $e")
analysisException.setStackTrace(e.getStackTrace)
throw analysisException
}
}
}

/**
* Construct a [[FunctionBuilder]] based on the provided class that represents a function.
*
* This performs reflection to decide what type of [[Expression]] to return in the builder.
*/
protected def makeFunctionBuilder(name: String, functionClassName: String): FunctionBuilder = {
// TODO: at least support UDAFs here
throw new UnsupportedOperationException("Use sqlContext.udf.register(...) instead.")
protected def makeFunctionExpression(
Copy link
Contributor

Choose a reason for hiding this comment

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

seems we need to catch exception for this method anyway, how about we just make this method return Expression and document that it can throw exception if the given class is not supported? Then HiveSessionCatalog can define its own exception message.

name: String,
clazz: Class[_],
children: Seq[Expression]): Option[Expression] = {
val clsForUDAF =
Utils.classForName("org.apache.spark.sql.expressions.UserDefinedAggregateFunction")
if (clsForUDAF.isAssignableFrom(clazz)) {
val cls = Utils.classForName("org.apache.spark.sql.execution.aggregate.ScalaUDAF")
Some(cls.getConstructor(classOf[Seq[Expression]], clsForUDAF, classOf[Int], classOf[Int])
.newInstance(children, clazz.newInstance().asInstanceOf[Object], Int.box(1), Int.box(1))
.asInstanceOf[Expression])
} else {
None
}
}

/**
Expand All @@ -1121,7 +1168,14 @@ class SessionCatalog(
}
val info = new ExpressionInfo(funcDefinition.className, func.database.orNull, func.funcName)
val builder =
functionBuilder.getOrElse(makeFunctionBuilder(func.unquotedString, funcDefinition.className))
functionBuilder.getOrElse {
val className = funcDefinition.className
if (!Utils.classIsLoadable(className)) {
throw new AnalysisException(s"Can not load class '$className' when registering " +
s"the function '$func', please make sure it is on the classpath")
}
makeFunctionBuilder(func.unquotedString, className)
}
functionRegistry.registerFunction(func, info, builder)
}

Expand Down
13 changes: 13 additions & 0 deletions sql/core/src/test/resources/sql-tests/inputs/udaf.sql
Original file line number Diff line number Diff line change
@@ -0,0 +1,13 @@
CREATE OR REPLACE TEMPORARY VIEW t1 AS SELECT * FROM VALUES
(1), (2), (3), (4)
as t1(int_col1);

CREATE FUNCTION myDoubleAvg AS 'test.org.apache.spark.sql.MyDoubleAvg';

SELECT default.myDoubleAvg(int_col1) as my_avg from t1;

SELECT default.myDoubleAvg(int_col1, 3) as my_avg from t1;

CREATE FUNCTION udaf1 AS 'test.non.existent.udaf';

SELECT default.udaf1(int_col1) as udaf1 from t1;
54 changes: 54 additions & 0 deletions sql/core/src/test/resources/sql-tests/results/udaf.sql.out
Original file line number Diff line number Diff line change
@@ -0,0 +1,54 @@
-- Automatically generated by SQLQueryTestSuite
-- Number of queries: 6


-- !query 0
CREATE OR REPLACE TEMPORARY VIEW t1 AS SELECT * FROM VALUES
(1), (2), (3), (4)
as t1(int_col1)
-- !query 0 schema
struct<>
-- !query 0 output



-- !query 1
CREATE FUNCTION myDoubleAvg AS 'test.org.apache.spark.sql.MyDoubleAvg'
-- !query 1 schema
struct<>
-- !query 1 output



-- !query 2
SELECT default.myDoubleAvg(int_col1) as my_avg from t1
-- !query 2 schema
struct<my_avg:double>
-- !query 2 output
102.5


-- !query 3
SELECT default.myDoubleAvg(int_col1, 3) as my_avg from t1
-- !query 3 schema
struct<>
-- !query 3 output
java.lang.AssertionError
assertion failed: Incorrect number of children


-- !query 4
CREATE FUNCTION udaf1 AS 'test.non.existent.udaf'
-- !query 4 schema
struct<>
-- !query 4 output



-- !query 5
SELECT default.udaf1(int_col1) as udaf1 from t1
-- !query 5 schema
struct<>
-- !query 5 output
org.apache.spark.sql.AnalysisException
Can not load class 'test.non.existent.udaf' when registering the function 'default.udaf1', please make sure it is on the classpath; line 1 pos 7
Original file line number Diff line number Diff line change
Expand Up @@ -27,7 +27,6 @@ import org.apache.hadoop.hive.ql.exec.{UDAF, UDF}
import org.apache.hadoop.hive.ql.exec.{FunctionRegistry => HiveFunctionRegistry}
import org.apache.hadoop.hive.ql.udf.generic.{AbstractGenericUDAFResolver, GenericUDF, GenericUDTF}

import org.apache.spark.sql.AnalysisException
import org.apache.spark.sql.catalyst.FunctionIdentifier
import org.apache.spark.sql.catalyst.analysis.FunctionRegistry
import org.apache.spark.sql.catalyst.analysis.FunctionRegistry.FunctionBuilder
Expand All @@ -37,7 +36,6 @@ import org.apache.spark.sql.catalyst.parser.ParserInterface
import org.apache.spark.sql.hive.HiveShim.HiveFunctionWrapper
import org.apache.spark.sql.internal.SQLConf
import org.apache.spark.sql.types.{DecimalType, DoubleType}
import org.apache.spark.util.Utils


private[sql] class HiveSessionCatalog(
Expand All @@ -58,54 +56,44 @@ private[sql] class HiveSessionCatalog(
parser,
functionResourceLoader) {

override def makeFunctionBuilder(funcName: String, className: String): FunctionBuilder = {
makeFunctionBuilder(funcName, Utils.classForName(className))
}

/**
* Construct a [[FunctionBuilder]] based on the provided class that represents a function.
*/
private def makeFunctionBuilder(name: String, clazz: Class[_]): FunctionBuilder = {
// When we instantiate hive UDF wrapper class, we may throw exception if the input
// expressions don't satisfy the hive UDF, such as type mismatch, input number
// mismatch, etc. Here we catch the exception and throw AnalysisException instead.
(children: Seq[Expression]) => {
try {
if (classOf[UDF].isAssignableFrom(clazz)) {
val udf = HiveSimpleUDF(name, new HiveFunctionWrapper(clazz.getName), children)
udf.dataType // Force it to check input data types.
udf
} else if (classOf[GenericUDF].isAssignableFrom(clazz)) {
val udf = HiveGenericUDF(name, new HiveFunctionWrapper(clazz.getName), children)
udf.dataType // Force it to check input data types.
udf
} else if (classOf[AbstractGenericUDAFResolver].isAssignableFrom(clazz)) {
val udaf = HiveUDAFFunction(name, new HiveFunctionWrapper(clazz.getName), children)
udaf.dataType // Force it to check input data types.
udaf
} else if (classOf[UDAF].isAssignableFrom(clazz)) {
val udaf = HiveUDAFFunction(
name,
new HiveFunctionWrapper(clazz.getName),
children,
isUDAFBridgeRequired = true)
udaf.dataType // Force it to check input data types.
udaf
} else if (classOf[GenericUDTF].isAssignableFrom(clazz)) {
val udtf = HiveGenericUDTF(name, new HiveFunctionWrapper(clazz.getName), children)
udtf.elementSchema // Force it to check input data types.
udtf
} else {
throw new AnalysisException(s"No handler for Hive UDF '${clazz.getCanonicalName}'")
}
} catch {
case ae: AnalysisException =>
throw ae
case NonFatal(e) =>
val analysisException =
new AnalysisException(s"No handler for Hive UDF '${clazz.getCanonicalName}': $e")
analysisException.setStackTrace(e.getStackTrace)
throw analysisException
override def makeFunctionExpression(
name: String,
clazz: Class[_],
children: Seq[Expression]): Option[Expression] = {

super.makeFunctionExpression(name, clazz, children).orElse {
// When we instantiate hive UDF wrapper class, we may throw exception if the input
// expressions don't satisfy the hive UDF, such as type mismatch, input number
// mismatch, etc. Here we catch the exception and throw AnalysisException instead.
if (classOf[UDF].isAssignableFrom(clazz)) {
val udf = HiveSimpleUDF(name, new HiveFunctionWrapper(clazz.getName), children)
udf.dataType // Force it to check input data types.
Some(udf)
} else if (classOf[GenericUDF].isAssignableFrom(clazz)) {
val udf = HiveGenericUDF(name, new HiveFunctionWrapper(clazz.getName), children)
udf.dataType // Force it to check input data types.
Some(udf)
} else if (classOf[AbstractGenericUDAFResolver].isAssignableFrom(clazz)) {
val udaf = HiveUDAFFunction(name, new HiveFunctionWrapper(clazz.getName), children)
udaf.dataType // Force it to check input data types.
Some(udaf)
} else if (classOf[UDAF].isAssignableFrom(clazz)) {
val udaf = HiveUDAFFunction(
name,
new HiveFunctionWrapper(clazz.getName),
children,
isUDAFBridgeRequired = true)
udaf.dataType // Force it to check input data types.
Some(udaf)
} else if (classOf[GenericUDTF].isAssignableFrom(clazz)) {
val udtf = HiveGenericUDTF(name, new HiveFunctionWrapper(clazz.getName), children)
udtf.elementSchema // Force it to check input data types.
Some(udtf)
} else {
None
}
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,6 +26,7 @@ import org.apache.hadoop.hive.ql.util.JavaDataModel
import org.apache.hadoop.hive.serde2.objectinspector.{ObjectInspector, ObjectInspectorFactory}
import org.apache.hadoop.hive.serde2.objectinspector.primitive.PrimitiveObjectInspectorFactory
import org.apache.hadoop.hive.serde2.typeinfo.TypeInfo
import test.org.apache.spark.sql.MyDoubleAvg

import org.apache.spark.sql.{AnalysisException, QueryTest, Row}
import org.apache.spark.sql.execution.aggregate.ObjectHashAggregateExec
Expand Down Expand Up @@ -86,6 +87,18 @@ class HiveUDAFSuite extends QueryTest with TestHiveSingleton with SQLTestUtils {
))
}

test("call JAVA UDAF") {
withTempView("temp") {
withUserDefinedFunction("myDoubleAvg" -> false) {
spark.range(1, 10).toDF("value").createOrReplaceTempView("temp")
sql(s"CREATE FUNCTION myDoubleAvg AS '${classOf[MyDoubleAvg].getName}'")
checkAnswer(
spark.sql("SELECT default.myDoubleAvg(value) as my_avg from temp"),
Row(105.0))
}
}
}

test("non-deterministic children expressions of UDAF") {
withTempView("view1") {
spark.range(1).selectExpr("id as x", "id as y").createTempView("view1")
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -404,59 +404,34 @@ class HiveUDFSuite extends QueryTest with TestHiveSingleton with SQLTestUtils {
}

test("Hive UDFs with insufficient number of input arguments should trigger an analysis error") {
Seq((1, 2)).toDF("a", "b").createOrReplaceTempView("testUDF")
withTempView("testUDF") {
Seq((1, 2)).toDF("a", "b").createOrReplaceTempView("testUDF")

def testErrorMsgForFunc(funcName: String, className: String): Unit = {
withUserDefinedFunction(funcName -> true) {
sql(s"CREATE TEMPORARY FUNCTION $funcName AS '$className'")
val message = intercept[AnalysisException] {
sql(s"SELECT $funcName() FROM testUDF")
}.getMessage
assert(message.contains(s"No handler for UDF/UDAF/UDTF '$className'"))
}
}

{
// HiveSimpleUDF
sql(s"CREATE TEMPORARY FUNCTION testUDFTwoListList AS '${classOf[UDFTwoListList].getName}'")
val message = intercept[AnalysisException] {
sql("SELECT testUDFTwoListList() FROM testUDF")
}.getMessage
assert(message.contains("No handler for Hive UDF"))
sql("DROP TEMPORARY FUNCTION IF EXISTS testUDFTwoListList")
}
testErrorMsgForFunc("testUDFTwoListList", classOf[UDFTwoListList].getName)

{
// HiveGenericUDF
sql(s"CREATE TEMPORARY FUNCTION testUDFAnd AS '${classOf[GenericUDFOPAnd].getName}'")
val message = intercept[AnalysisException] {
sql("SELECT testUDFAnd() FROM testUDF")
}.getMessage
assert(message.contains("No handler for Hive UDF"))
sql("DROP TEMPORARY FUNCTION IF EXISTS testUDFAnd")
}
testErrorMsgForFunc("testUDFAnd", classOf[GenericUDFOPAnd].getName)

{
// Hive UDAF
sql(s"CREATE TEMPORARY FUNCTION testUDAFPercentile AS '${classOf[UDAFPercentile].getName}'")
val message = intercept[AnalysisException] {
sql("SELECT testUDAFPercentile(a) FROM testUDF GROUP BY b")
}.getMessage
assert(message.contains("No handler for Hive UDF"))
sql("DROP TEMPORARY FUNCTION IF EXISTS testUDAFPercentile")
}
testErrorMsgForFunc("testUDAFPercentile", classOf[UDAFPercentile].getName)

{
// AbstractGenericUDAFResolver
sql(s"CREATE TEMPORARY FUNCTION testUDAFAverage AS '${classOf[GenericUDAFAverage].getName}'")
val message = intercept[AnalysisException] {
sql("SELECT testUDAFAverage() FROM testUDF GROUP BY b")
}.getMessage
assert(message.contains("No handler for Hive UDF"))
sql("DROP TEMPORARY FUNCTION IF EXISTS testUDAFAverage")
}
testErrorMsgForFunc("testUDAFAverage", classOf[GenericUDAFAverage].getName)

{
// Hive UDTF
sql(s"CREATE TEMPORARY FUNCTION testUDTFExplode AS '${classOf[GenericUDTFExplode].getName}'")
val message = intercept[AnalysisException] {
sql("SELECT testUDTFExplode() FROM testUDF")
}.getMessage
assert(message.contains("No handler for Hive UDF"))
sql("DROP TEMPORARY FUNCTION IF EXISTS testUDTFExplode")
// AbstractGenericUDAFResolver
testErrorMsgForFunc("testUDTFExplode", classOf[GenericUDTFExplode].getName)
}

spark.catalog.dropTempView("testUDF")
}

test("Hive UDF in group by") {
Expand Down