This is an automated email from the ASF dual-hosted git repository.
ibzib pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/beam.git
The following commit(s) were added to refs/heads/master by this push:
new 314efa7 [BEAM-10925] Create wrapper for user function definitions.
new ca9e4d2 Merge pull request #13307 from ibzib/udf-wrapper
314efa7 is described below
commit 314efa755e0baf4162de406dc96d2c9c307457bd
Author: Kyle Weaver <[email protected]>
AuthorDate: Wed Nov 11 13:30:51 2020 -0800
[BEAM-10925] Create wrapper for user function definitions.
Also, resolve native SQL scalar functions by path (list) instead of full
name (string) to match UDTVF.
---
.../extensions/sql/zetasql/ZetaSQLPlannerImpl.java | 22 +++++------
.../sql/zetasql/translation/ConversionContext.java | 24 ++----------
.../zetasql/translation/ExpressionConverter.java | 10 ++---
.../translation/UserFunctionDefinitions.java | 43 ++++++++++++++++++++++
4 files changed, 61 insertions(+), 38 deletions(-)
diff --git
a/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/ZetaSQLPlannerImpl.java
b/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/ZetaSQLPlannerImpl.java
index 8431bf0..6259d12 100644
---
a/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/ZetaSQLPlannerImpl.java
+++
b/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/ZetaSQLPlannerImpl.java
@@ -35,8 +35,8 @@ import
org.apache.beam.sdk.extensions.sql.impl.QueryPlanner.QueryParameters;
import
org.apache.beam.sdk.extensions.sql.zetasql.translation.ConversionContext;
import
org.apache.beam.sdk.extensions.sql.zetasql.translation.ExpressionConverter;
import
org.apache.beam.sdk.extensions.sql.zetasql.translation.QueryStatementConverter;
+import
org.apache.beam.sdk.extensions.sql.zetasql.translation.UserFunctionDefinitions;
import
org.apache.beam.vendor.calcite.v1_20_0.com.google.common.collect.ImmutableList;
-import
org.apache.beam.vendor.calcite.v1_20_0.com.google.common.collect.ImmutableMap;
import
org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.adapter.java.JavaTypeFactory;
import
org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.plan.RelOptCluster;
import
org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.plan.RelOptPlanner;
@@ -51,6 +51,7 @@ import
org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.tools.Framework
import
org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.tools.Frameworks;
import org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.tools.Program;
import org.apache.beam.vendor.calcite.v1_20_0.org.apache.calcite.util.Util;
+import
org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableMap;
/** ZetaSQLPlannerImpl. */
@SuppressWarnings({
@@ -100,7 +101,8 @@ class ZetaSQLPlannerImpl {
SimpleCatalog catalog =
analyzer.createPopulatedCatalog(defaultSchemaPlus.getName(), options,
tables);
- ImmutableMap.Builder<String, ResolvedCreateFunctionStmt> udfBuilder =
ImmutableMap.builder();
+ ImmutableMap.Builder<List<String>, ResolvedCreateFunctionStmt> udfBuilder =
+ ImmutableMap.builder();
ImmutableMap.Builder<List<String>, ResolvedNode> udtvfBuilder =
ImmutableMap.builder();
ResolvedStatement statement;
@@ -109,13 +111,7 @@ class ZetaSQLPlannerImpl {
statement = analyzer.analyzeNextStatement(parseResumeLocation, options,
catalog);
if (statement.nodeKind() == RESOLVED_CREATE_FUNCTION_STMT) {
ResolvedCreateFunctionStmt createFunctionStmt =
(ResolvedCreateFunctionStmt) statement;
- // ResolvedCreateFunctionStmt does not include the full function name,
so build it here.
- String functionFullName =
- String.format(
- "%s:%s",
- SqlAnalyzer.USER_DEFINED_FUNCTIONS,
- String.join(".", createFunctionStmt.getNamePath()));
- udfBuilder.put(functionFullName, createFunctionStmt);
+ udfBuilder.put(createFunctionStmt.getNamePath(), createFunctionStmt);
} else if (statement.nodeKind() == RESOLVED_CREATE_TABLE_FUNCTION_STMT) {
ResolvedCreateTableFunctionStmt createTableFunctionStmt =
(ResolvedCreateTableFunctionStmt) statement;
@@ -134,10 +130,12 @@ class ZetaSQLPlannerImpl {
"Statement list must end in a SELECT statement, not " +
statement.nodeKindString());
}
+ UserFunctionDefinitions userFunctionDefinitions =
+ new UserFunctionDefinitions(udfBuilder.build(), udtvfBuilder.build());
+
ExpressionConverter expressionConverter =
- new ExpressionConverter(cluster, params, udfBuilder.build());
- ConversionContext context =
- ConversionContext.of(config, expressionConverter, cluster, trait,
udtvfBuilder.build());
+ new ExpressionConverter(cluster, params, userFunctionDefinitions);
+ ConversionContext context = ConversionContext.of(config,
expressionConverter, cluster, trait);
RelNode convertedNode =
QueryStatementConverter.convertRootQuery(context, (ResolvedQueryStmt)
statement);
diff --git
a/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/ConversionContext.java
b/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/ConversionContext.java
index 781dac8..9523b76 100644
---
a/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/ConversionContext.java
+++
b/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/ConversionContext.java
@@ -18,7 +18,6 @@
package org.apache.beam.sdk.extensions.sql.zetasql.translation;
import com.google.zetasql.resolvedast.ResolvedNode;
-import java.util.Collections;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
@@ -36,11 +35,6 @@ public class ConversionContext {
private final RelOptCluster cluster;
private final QueryTrait trait;
- // SQL native user-defined table-valued function can be resolved by
Analyzer. Keeping the
- // function name to its ResolvedNode mapping so during Plan conversion,
UDTVF implementation
- // can replace inputs of TVFScanConverter.
- private final Map<List<String>, ResolvedNode>
userDefinedTableValuedFunctions;
-
// SQL native user-defined table-valued function can be resolved by
Analyzer. Its sql body is
// converted to ResolvedNode, in which function parameters are replaced with
ResolvedArgumentRef.
// Meanwhile, Analyzer provides values for function parameters because it
looks ahead to find
@@ -52,31 +46,19 @@ public class ConversionContext {
FrameworkConfig config,
ExpressionConverter expressionConverter,
RelOptCluster cluster,
- QueryTrait trait,
- Map<List<String>, ResolvedNode> sqlUDTVF) {
- return new ConversionContext(config, expressionConverter, cluster, trait,
sqlUDTVF);
- }
-
- public static ConversionContext of(
- FrameworkConfig config,
- ExpressionConverter expressionConverter,
- RelOptCluster cluster,
QueryTrait trait) {
- return new ConversionContext(
- config, expressionConverter, cluster, trait, Collections.emptyMap());
+ return new ConversionContext(config, expressionConverter, cluster, trait);
}
private ConversionContext(
FrameworkConfig config,
ExpressionConverter expressionConverter,
RelOptCluster cluster,
- QueryTrait trait,
- Map<List<String>, ResolvedNode> sqlUDTVF) {
+ QueryTrait trait) {
this.config = config;
this.expressionConverter = expressionConverter;
this.cluster = cluster;
this.trait = trait;
- this.userDefinedTableValuedFunctions = sqlUDTVF;
this.functionArgumentRefMapping = new HashMap<>();
}
@@ -97,7 +79,7 @@ public class ConversionContext {
}
Map<List<String>, ResolvedNode> getUserDefinedTableValuedFunctions() {
- return userDefinedTableValuedFunctions;
+ return
getExpressionConverter().userFunctionDefinitions.sqlTableValuedFunctions;
}
Map<String, RexNode> getFunctionArgumentRefMapping() {
diff --git
a/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/ExpressionConverter.java
b/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/ExpressionConverter.java
index db5b114..24a5e1c 100644
---
a/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/ExpressionConverter.java
+++
b/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/ExpressionConverter.java
@@ -162,15 +162,15 @@ public class ExpressionConverter {
private final RelOptCluster cluster;
private final QueryParameters queryParams;
private int nullParamCount = 0;
- private final Map<String, ResolvedCreateFunctionStmt> userDefinedFunctions;
+ final UserFunctionDefinitions userFunctionDefinitions;
public ExpressionConverter(
RelOptCluster cluster,
QueryParameters params,
- Map<String, ResolvedCreateFunctionStmt> userDefinedFunctions) {
+ UserFunctionDefinitions userFunctionDefinitions) {
this.cluster = cluster;
this.queryParams = params;
- this.userDefinedFunctions = userDefinedFunctions;
+ this.userFunctionDefinitions = userFunctionDefinitions;
}
/** Extract expressions from a project scan node. */
@@ -665,8 +665,8 @@ public class ExpressionConverter {
convertRexNodeFromResolvedExpr(expr, columnList, fieldList,
outerFunctionArguments));
}
} else if (funGroup.equals(USER_DEFINED_FUNCTIONS)) {
- String fullName = functionCall.getFunction().getFullName();
- ResolvedCreateFunctionStmt createFunctionStmt =
userDefinedFunctions.get(fullName);
+ ResolvedCreateFunctionStmt createFunctionStmt =
+
userFunctionDefinitions.sqlScalarFunctions.get(functionCall.getFunction().getNamePath());
ResolvedExpr functionExpression =
createFunctionStmt.getFunctionExpression();
ImmutableMap.Builder<String, RexNode> innerFunctionArguments =
ImmutableMap.builder();
for (int i = 0; i < functionCall.getArgumentList().size(); i++) {
diff --git
a/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/UserFunctionDefinitions.java
b/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/UserFunctionDefinitions.java
new file mode 100644
index 0000000..a4544e7
--- /dev/null
+++
b/sdks/java/extensions/sql/zetasql/src/main/java/org/apache/beam/sdk/extensions/sql/zetasql/translation/UserFunctionDefinitions.java
@@ -0,0 +1,43 @@
+/*
+ * 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.beam.sdk.extensions.sql.zetasql.translation;
+
+import com.google.zetasql.resolvedast.ResolvedNode;
+import com.google.zetasql.resolvedast.ResolvedNodes;
+import java.util.List;
+import
org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableMap;
+
+/** Holds user defined function definitions. */
+public class UserFunctionDefinitions {
+ public final ImmutableMap<List<String>,
ResolvedNodes.ResolvedCreateFunctionStmt>
+ sqlScalarFunctions;
+
+ /**
+ * SQL native user-defined table-valued function can be resolved by
Analyzer. Keeping the function
+ * name to its ResolvedNode mapping so during Plan conversion, UDTVF
implementation can replace
+ * inputs of TVFScanConverter.
+ */
+ public final ImmutableMap<List<String>, ResolvedNode>
sqlTableValuedFunctions;
+
+ public UserFunctionDefinitions(
+ ImmutableMap<List<String>, ResolvedNodes.ResolvedCreateFunctionStmt>
sqlScalarFunctions,
+ ImmutableMap<List<String>, ResolvedNode> sqlTableValuedFunctions) {
+ this.sqlScalarFunctions = sqlScalarFunctions;
+ this.sqlTableValuedFunctions = sqlTableValuedFunctions;
+ }
+}