This is an automated email from the ASF dual-hosted git repository.
lzljs3620320 pushed a commit to branch release-1.13
in repository https://gitbox.apache.org/repos/asf/flink.git
The following commit(s) were added to refs/heads/release-1.13 by this push:
new 7dee2c2 [FLINK-22092][hive] Ignore static conf file URLs in HiveConf
7dee2c2 is described below
commit 7dee2c2b90118eaca1a0c41046f89f93e4d4c6c7
Author: Rui Li <[email protected]>
AuthorDate: Wed Apr 21 20:41:39 2021 +0800
[FLINK-22092][hive] Ignore static conf file URLs in HiveConf
This closes #15692
---
.../apache/flink/table/catalog/hive/HiveCatalog.java | 20 +++++++++++++++++---
.../flink/table/catalog/hive/HiveCatalogTest.java | 17 +++++++++++++++++
.../flink/table/catalog/hive/HiveTestUtils.java | 5 ++---
.../src/test/resources/hive-site.xml | 5 +++++
4 files changed, 41 insertions(+), 6 deletions(-)
diff --git
a/flink-connectors/flink-connector-hive/src/main/java/org/apache/flink/table/catalog/hive/HiveCatalog.java
b/flink-connectors/flink-connector-hive/src/main/java/org/apache/flink/table/catalog/hive/HiveCatalog.java
index 87c41e8..ccaeb00 100644
---
a/flink-connectors/flink-connector-hive/src/main/java/org/apache/flink/table/catalog/hive/HiveCatalog.java
+++
b/flink-connectors/flink-connector-hive/src/main/java/org/apache/flink/table/catalog/hive/HiveCatalog.java
@@ -108,6 +108,7 @@ import java.io.File;
import java.io.FileNotFoundException;
import java.io.IOException;
import java.io.InputStream;
+import java.net.URL;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
@@ -145,6 +146,7 @@ public class HiveCatalog extends AbstractCatalog {
public static final String DEFAULT_DB = "default";
private static final Logger LOG =
LoggerFactory.getLogger(HiveCatalog.class);
+ public static final String HIVE_SITE_FILE = "hive-site.xml";
// Prefix used to distinguish scala/python functions
private static final String FLINK_SCALA_FUNCTION_PREFIX = "flink:scala:";
@@ -217,8 +219,8 @@ public class HiveCatalog extends AbstractCatalog {
LOG.info("Created HiveCatalog '{}'", catalogName);
}
- private static HiveConf createHiveConf(
- @Nullable String hiveConfDir, @Nullable String hadoopConfDir) {
+ @VisibleForTesting
+ static HiveConf createHiveConf(@Nullable String hiveConfDir, @Nullable
String hadoopConfDir) {
// create HiveConf from hadoop configuration with hadoop conf
directory configured.
Configuration hadoopConf = null;
if (isNullOrWhitespaceOnly(hadoopConfDir)) {
@@ -246,12 +248,16 @@ public class HiveCatalog extends AbstractCatalog {
if (hadoopConf == null) {
hadoopConf = new Configuration();
}
+ // ignore all the static conf file URLs that HiveConf may have set
+ HiveConf.setHiveSiteLocation(null);
+ HiveConf.setLoadMetastoreConfig(false);
+ HiveConf.setLoadHiveServer2Config(false);
HiveConf hiveConf = new HiveConf(hadoopConf, HiveConf.class);
LOG.info("Setting hive conf dir as {}", hiveConfDir);
if (hiveConfDir != null) {
- Path hiveSite = new Path(hiveConfDir, "hive-site.xml");
+ Path hiveSite = new Path(hiveConfDir, HIVE_SITE_FILE);
if (!hiveSite.toUri().isAbsolute()) {
// treat relative URI as local file to be compatible with
previous behavior
hiveSite = new Path(new File(hiveSite.toString()).toURI());
@@ -264,6 +270,14 @@ public class HiveCatalog extends AbstractCatalog {
throw new CatalogException(
"Failed to load hive-site.xml from specified path:" +
hiveSite, e);
}
+ } else {
+ // user doesn't provide hive conf dir, we try to find it in
classpath
+ URL hiveSite =
+
Thread.currentThread().getContextClassLoader().getResource(HIVE_SITE_FILE);
+ if (hiveSite != null) {
+ LOG.info("Found {} in classpath: {}", HIVE_SITE_FILE,
hiveSite);
+ hiveConf.addResource(hiveSite);
+ }
}
return hiveConf;
}
diff --git
a/flink-connectors/flink-connector-hive/src/test/java/org/apache/flink/table/catalog/hive/HiveCatalogTest.java
b/flink-connectors/flink-connector-hive/src/test/java/org/apache/flink/table/catalog/hive/HiveCatalogTest.java
index 0560364..b6e0f26 100644
---
a/flink-connectors/flink-connector-hive/src/test/java/org/apache/flink/table/catalog/hive/HiveCatalogTest.java
+++
b/flink-connectors/flink-connector-hive/src/test/java/org/apache/flink/table/catalog/hive/HiveCatalogTest.java
@@ -29,6 +29,7 @@ import org.apache.flink.table.catalog.hive.util.HiveTableUtil;
import org.apache.flink.table.descriptors.FileSystem;
import org.apache.flink.table.factories.FactoryUtil;
+import org.apache.hadoop.hive.conf.HiveConf;
import org.apache.hadoop.hive.metastore.api.Table;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -40,6 +41,7 @@ import java.util.Map;
import static org.apache.flink.table.factories.FactoryUtil.CONNECTOR;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertNull;
import static org.junit.Assert.assertTrue;
/** Test for HiveCatalog. */
@@ -120,4 +122,19 @@ public class HiveCatalogTest {
assertEquals(
hiveTable.getOptions().get("flink.url"),
"jdbc:clickhouse://host:port/testUrl2");
}
+
+ @Test
+ public void testCreateHiveConf() {
+ // hive-conf-dir not specified, should read hive-site from classpath
+ HiveConf hiveConf = HiveCatalog.createHiveConf(null, null);
+ assertEquals("common-val", hiveConf.get("common-key"));
+ // hive-conf-dir specified, shouldn't read hive-site from classpath
+ String hiveConfDir =
+ Thread.currentThread()
+ .getContextClassLoader()
+ .getResource("test-catalog-factory-conf")
+ .getPath();
+ hiveConf = HiveCatalog.createHiveConf(hiveConfDir, null);
+ assertNull(hiveConf.get("common-key", null));
+ }
}
diff --git
a/flink-connectors/flink-connector-hive/src/test/java/org/apache/flink/table/catalog/hive/HiveTestUtils.java
b/flink-connectors/flink-connector-hive/src/test/java/org/apache/flink/table/catalog/hive/HiveTestUtils.java
index ab0c210..802f3d1 100644
---
a/flink-connectors/flink-connector-hive/src/test/java/org/apache/flink/table/catalog/hive/HiveTestUtils.java
+++
b/flink-connectors/flink-connector-hive/src/test/java/org/apache/flink/table/catalog/hive/HiveTestUtils.java
@@ -59,7 +59,6 @@ import static
org.apache.flink.table.api.config.ExecutionConfigOptions.TABLE_EXE
/** Test utils for Hive connector. */
public class HiveTestUtils {
- private static final String HIVE_SITE_XML = "hive-site.xml";
private static final String HIVE_WAREHOUSE_URI_FORMAT =
"jdbc:derby:;databaseName=%s;create=true";
private static final TemporaryFolder TEMPORARY_FOLDER = new
TemporaryFolder();
@@ -111,8 +110,7 @@ public class HiveTestUtils {
}
public static HiveConf createHiveConf() {
- ClassLoader classLoader = new
HiveTestUtils().getClass().getClassLoader();
- HiveConf.setHiveSiteLocation(classLoader.getResource(HIVE_SITE_XML));
+ ClassLoader classLoader = HiveTestUtils.class.getClassLoader();
try {
TEMPORARY_FOLDER.create();
@@ -120,6 +118,7 @@ public class HiveTestUtils {
String warehouseUri = String.format(HIVE_WAREHOUSE_URI_FORMAT,
warehouseDir);
HiveConf hiveConf = new HiveConf();
+
hiveConf.addResource(classLoader.getResource(HiveCatalog.HIVE_SITE_FILE));
hiveConf.setVar(
HiveConf.ConfVars.METASTOREWAREHOUSE,
TEMPORARY_FOLDER.newFolder("hive_warehouse").getAbsolutePath());
diff --git
a/flink-connectors/flink-connector-hive/src/test/resources/hive-site.xml
b/flink-connectors/flink-connector-hive/src/test/resources/hive-site.xml
index c83bab8..cf7c80a 100644
--- a/flink-connectors/flink-connector-hive/src/test/resources/hive-site.xml
+++ b/flink-connectors/flink-connector-hive/src/test/resources/hive-site.xml
@@ -39,4 +39,9 @@
<value>true</value>
</property>
+ <property>
+ <name>common-key</name>
+ <value>common-val</value>
+ </property>
+
</configuration>