[ 
https://issues.apache.org/jira/browse/BEAM-5644?focusedWorklogId=195516&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-195516
 ]

ASF GitHub Bot logged work on BEAM-5644:
----------------------------------------

                Author: ASF GitHub Bot
            Created on: 07/Feb/19 07:04
            Start Date: 07/Feb/19 07:04
    Worklog Time Spent: 10m 
      Work Description: amaliujia commented on pull request #7745: [BEAM-5644] 
make Planner configurable
URL: https://github.com/apache/beam/pull/7745#discussion_r254564728
 
 

 ##########
 File path: 
sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamSqlEnv.java
 ##########
 @@ -174,8 +106,110 @@ public void executeDdl(String sqlStatement) throws 
ParseException {
   public String explain(String sqlString) throws ParseException {
     try {
       return RelOptUtil.toString(planner.convertToBeamRel(sqlString));
-    } catch (ValidationException | RelConversionException | SqlParseException 
e) {
+    } catch (Exception e) {
       throw new ParseException("Unable to parse statement", e);
     }
   }
+
+  /** BeamSqlEnv's Builder. */
+  public static class BeamSqlEnvBuilder {
+    private final JdbcConnection jdbcConnection;
+    private String plannerName = "CalcitePlanner";
+
+    public static BeamSqlEnvBuilder builder(TableProvider tableProvider) {
+      return new BeamSqlEnvBuilder(tableProvider);
+    }
+
+    private BeamSqlEnvBuilder(TableProvider tableProvider) {
+      jdbcConnection = JdbcDriver.connect(tableProvider);
+    }
+
+    public BeamSqlEnvBuilder registerBuiltinUdf(Map<String, List<Method>> 
methods) {
+      for (Map.Entry<String, List<Method>> entry : methods.entrySet()) {
+        for (Method method : entry.getValue()) {
+          jdbcConnection.getCurrentSchemaPlus().add(entry.getKey(), 
UdfImpl.create(method));
+        }
+      }
+
+      return this;
+    }
+
+    public BeamSqlEnvBuilder addSchema(String name, TableProvider 
tableProvider) {
+      jdbcConnection.setSchema(name, tableProvider);
+
+      return this;
+    }
+
+    public BeamSqlEnvBuilder setCurrentSchema(String name) {
+      try {
+        jdbcConnection.setSchema(name);
+      } catch (SQLException e) {
+        throw new RuntimeException(e);
+      }
+
+      return this;
+    }
+
+    /** Register a UDF function which can be used in SQL expression. */
+    public BeamSqlEnvBuilder registerUdf(String functionName, Class<?> clazz, 
String method) {
+      jdbcConnection.getCurrentSchemaPlus().add(functionName, 
UdfImpl.create(clazz, method));
+
+      return this;
+    }
+
+    /** Register a UDF function which can be used in SQL expression. */
+    public BeamSqlEnvBuilder registerUdf(String functionName, Class<? extends 
BeamSqlUdf> clazz) {
+      return registerUdf(functionName, clazz, BeamSqlUdf.UDF_METHOD);
+    }
+
+    public BeamSqlEnvBuilder registerUdf(String functionName, 
SerializableFunction sfn) {
+      return registerUdf(functionName, sfn.getClass(), "apply");
+    }
+
+    /**
+     * Register a UDAF function which can be used in GROUP-BY expression. See 
{@link
+     * org.apache.beam.sdk.transforms.Combine.CombineFn} on how to implement a 
UDAF.
+     */
+    public BeamSqlEnvBuilder registerUdaf(String functionName, 
Combine.CombineFn combineFn) {
+      jdbcConnection.getCurrentSchemaPlus().add(functionName, new 
UdafImpl(combineFn));
+
+      return this;
+    }
+
+    /** Load all UDF/UDAF from {@link UdfUdafProvider}. */
+    public BeamSqlEnvBuilder loadUdfUdafFromProvider() {
+      ServiceLoader.<UdfUdafProvider>load(UdfUdafProvider.class)
+          .forEach(
+              ins -> {
+                ins.getBeamSqlUdfs().forEach((udfName, udfClass) -> 
registerUdf(udfName, udfClass));
+                ins.getSerializableFunctionUdfs()
+                    .forEach((udfName, udfFn) -> registerUdf(udfName, udfFn));
+                ins.getUdafs().forEach((udafName, udafFn) -> 
registerUdaf(udafName, udafFn));
+              });
+
+      return this;
+    }
+
+    public BeamSqlEnvBuilder loadBeamBuiltinFunctions() {
+      for (BeamBuiltinFunctionProvider provider :
+          ServiceLoader.load(BeamBuiltinFunctionProvider.class)) {
+        registerBuiltinUdf(provider.getBuiltinMethods());
+      }
+
+      return this;
+    }
+
+    public BeamSqlEnvBuilder setPlannerName(String name) {
+      plannerName = name;
+      return this;
+    }
+
+    public BeamSqlEnv build() {
+      if (plannerName.equals("CalcitePlanner")) {
 
 Review comment:
   I would prefer to leave it for the future. 
   
   Right now there is only one planner anyway. It's less clear what constructor 
that new planner should have. I probably can add a list of try: try constructor 
one, not found, then try constructor two, and so on. But the list of 
constructor should be put there when we know they are useful (e.g. by seeing 
more planner). In order to make the reflection useful, I would hope to see 
someone has the use case to provide planner through class path and they are 
willing to contribute their way of constructor. 
   
   Also, for any planner that has checked in, they should be initialized 
directly.  
 
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
[email protected]


Issue Time Tracking
-------------------

    Worklog Id:     (was: 195516)
    Time Spent: 8h 40m  (was: 8.5h)

> make Planner configurable 
> --------------------------
>
>                 Key: BEAM-5644
>                 URL: https://issues.apache.org/jira/browse/BEAM-5644
>             Project: Beam
>          Issue Type: New Feature
>          Components: dsl-sql
>            Reporter: Rui Wang
>            Assignee: Rui Wang
>            Priority: Major
>              Labels: triaged
>          Time Spent: 8h 40m
>  Remaining Estimate: 0h
>
> We can make planner configurable here: 
> [BeamQueryPlanner.java#L145|https://github.com/apache/beam/blob/master/sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/BeamQueryPlanner.java#L145]
>  
> By doing so, we can have different planner implementation to support 
> different SQL dialect.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to