xushiyan commented on code in PR #13259:
URL: https://github.com/apache/hudi/pull/13259#discussion_r2072775651


##########
hudi-client/hudi-flink-client/src/main/java/org/apache/hudi/util/FlinkClientUtil.java:
##########
@@ -56,6 +67,22 @@ public static org.apache.hadoop.conf.Configuration 
getHadoopConf() {
     if (hadoopConf == null) {
       hadoopConf = new org.apache.hadoop.conf.Configuration();
     }
+
+    String confDir = System.getenv(FLINK_CONF_DIR);
+    if (StringUtils.isNullOrEmpty(confDir)) {
+      confDir = System.getenv(ApplicationConstants.Environment.PWD.key());
+    }
+
+    LOG.info("Flink conf dir: {}", confDir);
+    try {
+      Configuration configuration = 
GlobalConfiguration.loadConfiguration(confDir);
+      for (Map.Entry<String, String> kv : 
ConfigurationUtils.getPrefixedKeyValuePairs(HADOOP_CONFIG_PREFIX, 
configuration).entrySet()) {
+        hadoopConf.set(kv.getKey(), kv.getValue());
+      }
+    } catch (Exception e) {
+      Log.info("Fail to load flink configuration from path {}", confDir, e);

Review Comment:
   ```suggestion
         Log.warn("Fail to load flink configuration from path {}", confDir, e);
   ```



##########
hudi-client/hudi-flink-client/src/main/java/org/apache/hudi/util/FlinkClientUtil.java:
##########
@@ -56,6 +67,22 @@ public static org.apache.hadoop.conf.Configuration 
getHadoopConf() {
     if (hadoopConf == null) {
       hadoopConf = new org.apache.hadoop.conf.Configuration();
     }
+
+    String confDir = System.getenv(FLINK_CONF_DIR);
+    if (StringUtils.isNullOrEmpty(confDir)) {
+      confDir = System.getenv(ApplicationConstants.Environment.PWD.key());

Review Comment:
   if FLINK_CONF_DIR not specified by user, we can just skip it. Loading conf 
from  `ApplicationConstants.Environment.PWD` by default does not look desirable 
- is this a common known behavior?



##########
hudi-flink-datasource/hudi-flink/pom.xml:
##########
@@ -355,6 +355,13 @@
             <version>${project.version}</version>
             <scope>test</scope>
         </dependency>
+
+        <dependency>
+            <groupId>org.mockito</groupId>
+            <artifactId>mockito-inline</artifactId>
+            <scope>test</scope>
+        </dependency>

Review Comment:
   can this be added to tests-common so it can benefit all modules?



##########
hudi-client/hudi-flink-client/src/main/java/org/apache/hudi/util/FlinkClientUtil.java:
##########
@@ -19,18 +19,29 @@
 package org.apache.hudi.util;
 
 import org.apache.hudi.common.table.HoodieTableMetaClient;
+import org.apache.hudi.common.util.StringUtils;
 import org.apache.hudi.hadoop.fs.HadoopFSUtils;
 
 import org.apache.flink.api.java.hadoop.mapred.utils.HadoopUtils;
 import org.apache.flink.configuration.Configuration;
+import org.apache.flink.configuration.ConfigurationUtils;
+import org.apache.flink.configuration.GlobalConfiguration;
 import org.apache.hadoop.fs.Path;
+import org.apache.hadoop.yarn.api.ApplicationConstants;
+import org.mortbay.log.Log;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
 
 import java.io.File;
+import java.util.Map;
 
 /**
  * Utilities for Hoodie Flink client.
  */
 public class FlinkClientUtil {
+  private static final Logger LOG = 
LoggerFactory.getLogger(FlinkClientUtil.class);
+  public static String HADOOP_CONFIG_PREFIX = "flink.hadoop";

Review Comment:
   ```suggestion
     public static String FLINK_HADOOP_CONFIG_PREFIX = "flink.hadoop";
   ```



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to