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

pwason pushed a commit to branch release-0.14.0
in repository https://gitbox.apache.org/repos/asf/hudi.git

commit ed1d7c97d166edceeac77fdde15f39b2fb0b069f
Author: empcl <1515827...@qq.com>
AuthorDate: Tue Sep 5 10:24:34 2023 +0800

    [HUDI-6818] Create a database automatically when using the flink catalog 
dfs mode (#9592)
---
 .../main/java/org/apache/hudi/table/catalog/HoodieCatalog.java | 10 ++++++++++
 .../java/org/apache/hudi/table/catalog/TestHoodieCatalog.java  |  5 +++--
 2 files changed, 13 insertions(+), 2 deletions(-)

diff --git 
a/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/catalog/HoodieCatalog.java
 
b/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/catalog/HoodieCatalog.java
index 17e3cfa2838..d9e387476cb 100644
--- 
a/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/catalog/HoodieCatalog.java
+++ 
b/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/catalog/HoodieCatalog.java
@@ -125,6 +125,16 @@ public class HoodieCatalog extends AbstractCatalog {
     } catch (IOException e) {
       throw new CatalogException(String.format("Checking catalog path %s 
exists exception.", catalogPathStr), e);
     }
+
+    if (!databaseExists(getDefaultDatabase())) {
+      LOG.info("Creating database {} automatically because it does not 
exist.", getDefaultDatabase());
+      Path dbPath = new Path(catalogPath, getDefaultDatabase());
+      try {
+        fs.mkdirs(dbPath);
+      } catch (IOException e) {
+        throw new CatalogException(String.format("Creating database %s 
exception.", getDefaultDatabase()), e);
+      }
+    }
   }
 
   @Override
diff --git 
a/hudi-flink-datasource/hudi-flink/src/test/java/org/apache/hudi/table/catalog/TestHoodieCatalog.java
 
b/hudi-flink-datasource/hudi-flink/src/test/java/org/apache/hudi/table/catalog/TestHoodieCatalog.java
index 5983192fc82..dc4e0db058a 100644
--- 
a/hudi-flink-datasource/hudi-flink/src/test/java/org/apache/hudi/table/catalog/TestHoodieCatalog.java
+++ 
b/hudi-flink-datasource/hudi-flink/src/test/java/org/apache/hudi/table/catalog/TestHoodieCatalog.java
@@ -157,8 +157,9 @@ public class TestHoodieCatalog {
     streamTableEnv = TableEnvironmentImpl.create(settings);
     streamTableEnv.getConfig().getConfiguration()
         
.setInteger(ExecutionConfigOptions.TABLE_EXEC_RESOURCE_DEFAULT_PARALLELISM, 2);
-    File testDb = new File(tempFile, TEST_DEFAULT_DATABASE);
-    testDb.mkdir();
+
+    File catalogPath = new File(tempFile.getPath());
+    catalogPath.mkdir();
 
     catalog = new HoodieCatalog("hudi", 
Configuration.fromMap(getDefaultCatalogOption()));
     catalog.open();

Reply via email to