rdblue commented on a change in pull request #1558:
URL: https://github.com/apache/iceberg/pull/1558#discussion_r501878761
##########
File path: flink/src/main/java/org/apache/iceberg/flink/FlinkCatalogFactory.java
##########
@@ -121,4 +139,58 @@ protected Catalog createCatalog(String name, Map<String,
String> properties, Con
public static Configuration clusterHadoopConf() {
return
HadoopUtils.getHadoopConfiguration(GlobalConfiguration.loadConfiguration());
}
+
+ private void loadHiveConf(Configuration configuration, Map<String, String>
properties) {
+ String hiveConfPath = properties.get(HIVE_SITE_PATH);
+ Path path = new Path(hiveConfPath);
+ String scheme = getScheme(path);
+ // We can add more storage support later,like s3
+ switch (scheme) {
+ case HIVE_SITE_SCHEME_HDFS:
+ downloadFromHdfs(configuration, path);
+ break;
+ case HIVE_SITE_SCHEME_FILE:
+ loadLocalHiveConf(configuration, hiveConfPath);
+ break;
+ default:
+ throw new UnsupportedOperationException(
+ "Unsupported FileSystem for scheme :" + scheme);
+ }
+ }
+
+ private String getScheme(Path path) {
+ String scheme = path.toUri().getScheme();
+ if (scheme == null) {
+ // for case : /tmp/hive-site.xml
+ return HIVE_SITE_SCHEME_FILE;
+ } else {
+ return scheme;
+ }
+ }
+
+ private void loadLocalHiveConf(Configuration configuration, String
localHiveSitePath) {
+ File file = new File(localHiveSitePath);
+ if (!file.exists()) {
+ throw new RuntimeException(localHiveSitePath + " doesn't exist. if in
application mode ," +
+ " please provide a hdfs path for hive-site.xml");
+ } else {
+ configuration.addResource(localHiveSitePath);
+ }
+ }
+
+ private void downloadFromHdfs(Configuration configuration, Path
hdfsHiveSitePath) {
Review comment:
I've never seen code like this needed. Usually, `hive-site.xml` is
loaded from the classpath. Here's some code that does it from our Spark build:
```scala
val configFile = someClassLoader.getResource("hive-site.xml")
if (configFile != null) {
hadoopConfiguration.addResource(configFile)
}
```
----------------------------------------------------------------
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.
For queries about this service, please contact Infrastructure at:
[email protected]
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]