-
Notifications
You must be signed in to change notification settings - Fork 28.5k
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
[SPARK-21499] [SQL] Support creating persistent function for Spark UDAF(UserDefinedAggregateFunction) #18700
Closed
+204
−122
Closed
Changes from 4 commits
Commits
Show all changes
13 commits
Select commit
Hold shift + click to select a range
4028155
support UDAF
gatorsmile f634043
Merge remote-tracking branch 'upstream/master' into javaUDFinScala
gatorsmile a65607c
fix.
gatorsmile 12cefc2
fix.
gatorsmile bd5ae26
fix.
gatorsmile 7251be9
fix.
gatorsmile d3fbdc5
typo
gatorsmile 57607b5
fix.
gatorsmile 05e8168
fix.
gatorsmile aff8f9e
fix.
gatorsmile 7d9aabd
fix.
gatorsmile 8ea4ad1
Merge remote-tracking branch 'upstream/master' into pr-18700
gatorsmile 50224a7
fix.
gatorsmile File filter
Filter by extension
Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -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 | ||
|
@@ -40,6 +42,7 @@ import org.apache.spark.sql.catalyst.plans.logical.{LogicalPlan, SubqueryAlias, | |
import org.apache.spark.sql.catalyst.util.StringUtils | ||
import org.apache.spark.sql.internal.SQLConf | ||
import org.apache.spark.sql.types.StructType | ||
import org.apache.spark.util.Utils | ||
|
||
object SessionCatalog { | ||
val DEFAULT_DATABASE = "default" | ||
|
@@ -1096,8 +1099,42 @@ class SessionCatalog( | |
* 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.") | ||
makeFunctionBuilder(name, Utils.classForName(functionClassName)) | ||
} | ||
|
||
/** | ||
* Construct a [[FunctionBuilder]] based on the provided class that represents a function. | ||
*/ | ||
private def makeFunctionBuilder(name: String, clazz: Class[_]): FunctionBuilder = { | ||
// When we instantiate ScalaUDAF class, we may throw exception if the input | ||
// expressions don't satisfy the UDAF, such as type mismatch, input number | ||
// mismatch, etc. Here we catch the exception and throw AnalysisException instead. | ||
(children: Seq[Expression]) => { | ||
try { | ||
val clsForUDAF = | ||
Utils.classForName("org.apache.spark.sql.expressions.UserDefinedAggregateFunction") | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. shall we move the UDAF interface to catalyst? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. /**
* The base class for implementing user-defined aggregate functions (UDAF).
*
* @since 1.5.0
*/
@InterfaceStability.Stable
abstract class UserDefinedAggregateFunction This interface has been marked as stable. Can we still move it? or make a trait in Catalyst? |
||
if (clsForUDAF.isAssignableFrom(clazz)) { | ||
val cls = Utils.classForName("org.apache.spark.sql.execution.aggregate.ScalaUDAF") | ||
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 { | ||
throw new UnsupportedOperationException("Use sqlContext.udf.register(...) instead.") | ||
} | ||
} catch { | ||
case NonFatal(exception) => | ||
val e = exception match { | ||
// Since we are using shim, the exceptions thrown by the underlying method of | ||
// Method.invoke() are wrapped by InvocationTargetException | ||
case i: InvocationTargetException => i.getCause | ||
case o => o | ||
} | ||
val analysisException = | ||
new AnalysisException(s"No handler for UDAF '${clazz.getCanonicalName}': $e") | ||
analysisException.setStackTrace(e.getStackTrace) | ||
throw analysisException | ||
} | ||
} | ||
} | ||
|
||
/** | ||
|
@@ -1121,7 +1158,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) | ||
} | ||
|
||
|
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
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
54
sql/core/src/test/resources/sql-tests/results/udaf.sql.out
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
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 |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Add this suggestion to a batch that can be applied as a single commit.
This suggestion is invalid because no changes were made to the code.
Suggestions cannot be applied while the pull request is closed.
Suggestions cannot be applied while viewing a subset of changes.
Only one suggestion per line can be applied in a batch.
Add this suggestion to a batch that can be applied as a single commit.
Applying suggestions on deleted lines is not supported.
You must change the existing code in this line in order to create a valid suggestion.
Outdated suggestions cannot be applied.
This suggestion has been applied or marked resolved.
Suggestions cannot be applied from pending reviews.
Suggestions cannot be applied on multi-line comments.
Suggestions cannot be applied while the pull request is queued to merge.
Suggestion cannot be applied right now. Please check back later.
There was a problem hiding this comment.
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?There was a problem hiding this comment.
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 inHiveUDAFSuite.scala
to verify it.