You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by ja...@apache.org on 2022/09/21 01:46:49 UTC

[flink] 02/04: [FLINK-29045][hive] Optimize error message in Flink SQL Client and Gateway when try to use Hive Dialect

This is an automated email from the ASF dual-hosted git repository.

jark pushed a commit to branch release-1.16
in repository https://gitbox.apache.org/repos/asf/flink.git

commit 9e16d54b9ea0422a97bcbe20ebb244be54dc1c3c
Author: yuxia Luo <lu...@alumni.sjtu.edu.cn>
AuthorDate: Tue Sep 20 22:28:35 2022 +0800

    [FLINK-29045][hive] Optimize error message in Flink SQL Client and Gateway when try to use Hive Dialect
    
    This closes #20695
---
 .../client/gateway/context/SessionContext.java     | 11 +++++++
 .../gateway/service/context/SessionContext.java    | 34 +++++++++++++++-------
 2 files changed, 35 insertions(+), 10 deletions(-)

diff --git a/flink-table/flink-sql-client/src/main/java/org/apache/flink/table/client/gateway/context/SessionContext.java b/flink-table/flink-sql-client/src/main/java/org/apache/flink/table/client/gateway/context/SessionContext.java
index 7e44b65dad6..ef5917ea740 100644
--- a/flink-table/flink-sql-client/src/main/java/org/apache/flink/table/client/gateway/context/SessionContext.java
+++ b/flink-table/flink-sql-client/src/main/java/org/apache/flink/table/client/gateway/context/SessionContext.java
@@ -24,6 +24,8 @@ import org.apache.flink.configuration.ConfigOptions;
 import org.apache.flink.configuration.Configuration;
 import org.apache.flink.configuration.ReadableConfig;
 import org.apache.flink.table.api.EnvironmentSettings;
+import org.apache.flink.table.api.SqlDialect;
+import org.apache.flink.table.api.ValidationException;
 import org.apache.flink.table.catalog.Catalog;
 import org.apache.flink.table.catalog.CatalogManager;
 import org.apache.flink.table.catalog.FunctionCatalog;
@@ -34,6 +36,7 @@ import org.apache.flink.table.client.resource.ClientResourceManager;
 import org.apache.flink.table.client.util.ClientClassloaderUtil;
 import org.apache.flink.table.client.util.ClientWrapperClassLoader;
 import org.apache.flink.table.module.ModuleManager;
+import org.apache.flink.util.ExceptionUtils;
 import org.apache.flink.util.TemporaryClassLoaderContext;
 
 import org.slf4j.Logger;
@@ -156,6 +159,14 @@ public class SessionContext {
         } catch (Exception e) {
             // get error and reset the key with old value
             resetSessionConfigurationToDefault(originConfiguration);
+            if (value.equalsIgnoreCase(SqlDialect.HIVE.name())
+                    && e instanceof ValidationException) {
+                String additionErrorMsg =
+                        "Note: if you want to use Hive dialect, "
+                                + "please first move the jar `flink-table-planner_2.12` located in `FLINK_HOME/opt` "
+                                + "to `FLINK_HOME/lib` and then move out the jar `flink-table-planner-loader` from `FLINK_HOME/lib`.";
+                ExceptionUtils.updateDetailMessage(e, t -> t.getMessage() + additionErrorMsg);
+            }
             throw new SqlExecutionException(
                     String.format("Failed to set key %s with value %s.", key, value), e);
         }
diff --git a/flink-table/flink-sql-gateway/src/main/java/org/apache/flink/table/gateway/service/context/SessionContext.java b/flink-table/flink-sql-gateway/src/main/java/org/apache/flink/table/gateway/service/context/SessionContext.java
index 94cc9d13b32..67d73a6afc7 100644
--- a/flink-table/flink-sql-gateway/src/main/java/org/apache/flink/table/gateway/service/context/SessionContext.java
+++ b/flink-table/flink-sql-gateway/src/main/java/org/apache/flink/table/gateway/service/context/SessionContext.java
@@ -23,8 +23,10 @@ import org.apache.flink.configuration.ConfigOptions;
 import org.apache.flink.configuration.Configuration;
 import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
 import org.apache.flink.table.api.EnvironmentSettings;
+import org.apache.flink.table.api.SqlDialect;
 import org.apache.flink.table.api.TableConfig;
 import org.apache.flink.table.api.TableException;
+import org.apache.flink.table.api.ValidationException;
 import org.apache.flink.table.api.bridge.java.internal.StreamTableEnvironmentImpl;
 import org.apache.flink.table.api.config.TableConfigOptions;
 import org.apache.flink.table.api.internal.TableEnvironmentInternal;
@@ -46,6 +48,7 @@ import org.apache.flink.table.gateway.service.operation.OperationManager;
 import org.apache.flink.table.gateway.service.utils.SqlExecutionException;
 import org.apache.flink.table.module.ModuleManager;
 import org.apache.flink.table.resource.ResourceManager;
+import org.apache.flink.util.ExceptionUtils;
 import org.apache.flink.util.FlinkUserCodeClassLoaders;
 import org.apache.flink.util.MutableURLClassLoader;
 
@@ -293,16 +296,27 @@ public class SessionContext {
                         catalogManager,
                         functionCatalog);
 
-        return new StreamTableEnvironmentImpl(
-                catalogManager,
-                moduleManager,
-                resourceManager,
-                functionCatalog,
-                tableConfig,
-                env,
-                planner,
-                executor,
-                settings.isStreamingMode());
+        try {
+            return new StreamTableEnvironmentImpl(
+                    catalogManager,
+                    moduleManager,
+                    resourceManager,
+                    functionCatalog,
+                    tableConfig,
+                    env,
+                    planner,
+                    executor,
+                    settings.isStreamingMode());
+        } catch (ValidationException e) {
+            if (tableConfig.getSqlDialect() == SqlDialect.HIVE) {
+                String additionErrorMsg =
+                        "Note: if you want to use Hive dialect, "
+                                + "please first move the jar `flink-table-planner_2.12` located in `FLINK_HOME/opt` "
+                                + "to `FLINK_HOME/lib` and then move out the jar `flink-table-planner-loader` from `FLINK_HOME/lib`.";
+                ExceptionUtils.updateDetailMessage(e, t -> t.getMessage() + additionErrorMsg);
+            }
+            throw e;
+        }
     }
 
     private static Executor lookupExecutor(