This is an automated email from the ASF dual-hosted git repository.
yuqi4733 pushed a commit to branch branch-1.2
in repository https://gitbox.apache.org/repos/asf/gravitino.git
The following commit(s) were added to refs/heads/branch-1.2 by this push:
new a9bfa77b94 [Cherry-pick to branch-1.2] [#10741] fix(hive): Fix keytab
symlink TOCTOU race in FetchFileUtils (#10742) (#10786)
a9bfa77b94 is described below
commit a9bfa77b94803428652b5d3690df8fedfe2528b5
Author: github-actions[bot]
<41898282+github-actions[bot]@users.noreply.github.com>
AuthorDate: Wed Apr 15 23:05:28 2026 +0800
[Cherry-pick to branch-1.2] [#10741] fix(hive): Fix keytab symlink TOCTOU
race in FetchFileUtils (#10742) (#10786)
**Cherry-pick Information:**
- Original commit: 89ade4f6e3db703b634ed813c005e4da4c303baf
- Target branch: `branch-1.2`
- Status: ✅ Clean cherry-pick (no conflicts)
Co-authored-by: Yuhui <[email protected]>
---
.../gravitino/catalog/hive/FetchFileUtils.java | 65 -----------
.../gravitino/catalog/hive/TestFetchFileUtils.java | 95 ----------------
.../gravitino/hive/kerberos/FetchFileUtils.java | 45 +++++++-
.../gravitino/hive/kerberos/KerberosClient.java | 5 +-
.../hive/kerberos/TestFetchFileUtils.java | 119 +++++++++++++++++++++
5 files changed, 162 insertions(+), 167 deletions(-)
diff --git
a/catalogs/catalog-hive/src/main/java/org/apache/gravitino/catalog/hive/FetchFileUtils.java
b/catalogs/catalog-hive/src/main/java/org/apache/gravitino/catalog/hive/FetchFileUtils.java
deleted file mode 100644
index 527a3af043..0000000000
---
a/catalogs/catalog-hive/src/main/java/org/apache/gravitino/catalog/hive/FetchFileUtils.java
+++ /dev/null
@@ -1,65 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one
- * or more contributor license agreements. See the NOTICE file
- * distributed with this work for additional information
- * regarding copyright ownership. The ASF licenses this file
- * to you under the Apache License, Version 2.0 (the
- * "License"); you may not use this file except in compliance
- * with the License. You may obtain a copy of the License at
- *
- * http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing,
- * software distributed under the License is distributed on an
- * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
- * KIND, either express or implied. See the License for the
- * specific language governing permissions and limitations
- * under the License.
- */
-package org.apache.gravitino.catalog.hive;
-
-import java.io.File;
-import java.io.IOException;
-import java.net.URI;
-import java.net.URISyntaxException;
-import java.nio.file.Files;
-import java.util.Optional;
-import org.apache.commons.io.FileUtils;
-import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.fs.FileSystem;
-import org.apache.hadoop.fs.Path;
-
-public class FetchFileUtils {
-
- private FetchFileUtils() {}
-
- public static void fetchFileFromUri(
- String fileUri, File destFile, int timeout, Configuration conf) throws
IOException {
- try {
- URI uri = new URI(fileUri);
- String scheme = Optional.ofNullable(uri.getScheme()).orElse("file");
-
- switch (scheme) {
- case "http":
- case "https":
- case "ftp":
- FileUtils.copyURLToFile(uri.toURL(), destFile, timeout * 1000,
timeout * 1000);
- break;
-
- case "file":
- Files.createSymbolicLink(destFile.toPath(), new
File(uri.getPath()).toPath());
- break;
-
- case "hdfs":
- FileSystem.get(conf).copyToLocalFile(new Path(uri), new
Path(destFile.toURI()));
- break;
-
- default:
- throw new IllegalArgumentException(
- String.format("Doesn't support the scheme %s", scheme));
- }
- } catch (URISyntaxException ue) {
- throw new IllegalArgumentException("The uri of file has the wrong
format", ue);
- }
- }
-}
diff --git
a/catalogs/catalog-hive/src/test/java/org/apache/gravitino/catalog/hive/TestFetchFileUtils.java
b/catalogs/catalog-hive/src/test/java/org/apache/gravitino/catalog/hive/TestFetchFileUtils.java
deleted file mode 100644
index 4e7e3925ab..0000000000
---
a/catalogs/catalog-hive/src/test/java/org/apache/gravitino/catalog/hive/TestFetchFileUtils.java
+++ /dev/null
@@ -1,95 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one
- * or more contributor license agreements. See the NOTICE file
- * distributed with this work for additional information
- * regarding copyright ownership. The ASF licenses this file
- * to you under the Apache License, Version 2.0 (the
- * "License"); you may not use this file except in compliance
- * with the License. You may obtain a copy of the License at
- *
- * http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing,
- * software distributed under the License is distributed on an
- * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
- * KIND, either express or implied. See the License for the
- * specific language governing permissions and limitations
- * under the License.
- */
-
-package org.apache.gravitino.catalog.hive;
-
-import java.io.File;
-import java.io.IOException;
-import org.apache.hadoop.conf.Configuration;
-import org.junit.jupiter.api.Assertions;
-import org.junit.jupiter.api.Disabled;
-import org.junit.jupiter.api.Test;
-import org.slf4j.Logger;
-import org.slf4j.LoggerFactory;
-
-public class TestFetchFileUtils {
-
- private static final Logger LOG =
LoggerFactory.getLogger(TestFetchFileUtils.class);
- private static final int MAX_RETRIES = 8;
- private static final long INITIAL_RETRY_DELAY_MS = 1000;
-
- @Test
- public void testLinkLocalFile() throws Exception {
- File srcFile = new File("test");
- File destFile = new File("dest");
-
- try {
- if (srcFile.createNewFile()) {
- FetchFileUtils.fetchFileFromUri(
- srcFile.toURI().toString(), destFile, 10, new Configuration());
- Assertions.assertTrue(destFile.exists(), "Destination file should
exist after linking");
- } else {
- Assertions.fail("Failed to create the source file");
- }
- } finally {
- if (!srcFile.delete()) {
- LOG.warn("Failed to delete source file after test");
- }
- if (!destFile.delete()) {
- LOG.warn("Failed to delete destination file after test");
- }
- }
- }
-
- @Test
- @Disabled("The network errors in CI maybe cause the test failed")
- public void testDownloadFromHTTP() throws Exception {
- File destFile = new File("dest");
- String fileUrl = "https://downloads.apache.org/hadoop/common/KEYS";
- Configuration conf = new Configuration();
-
- boolean success = false;
- int attempts = 0;
-
- while (!success) {
- try {
- LOG.info("Attempting to download file from URL: {} (Attempt {})",
fileUrl, attempts + 1);
- FetchFileUtils.fetchFileFromUri(fileUrl, destFile, 45, conf);
- success = true;
- LOG.info("File downloaded successfully on attempt {}", attempts + 1);
- } catch (IOException e) {
- attempts++;
- LOG.error("Download attempt {} failed due to: {}", attempts,
e.getMessage(), e);
- if (attempts < MAX_RETRIES) {
- long retryDelay = INITIAL_RETRY_DELAY_MS * (1L << (attempts - 1));
- LOG.warn("Retrying in {} ms", retryDelay);
- Thread.sleep(retryDelay);
- } else {
- throw new AssertionError("Failed to download file after " +
MAX_RETRIES + " attempts", e);
- }
- }
- }
-
- Assertions.assertTrue(destFile.exists(), "File should exist after
successful download");
-
- if (!destFile.delete()) {
- LOG.warn("Failed to delete destination file after test");
- }
- }
-}
diff --git
a/catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/FetchFileUtils.java
b/catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/FetchFileUtils.java
index 743b1d562c..7149e1c82d 100644
---
a/catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/FetchFileUtils.java
+++
b/catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/FetchFileUtils.java
@@ -19,9 +19,13 @@
package org.apache.gravitino.hive.kerberos;
import java.io.File;
+import java.io.IOException;
import java.net.URI;
import java.net.URISyntaxException;
import java.nio.file.Files;
+import java.nio.file.StandardCopyOption;
+import java.util.Optional;
+import java.util.concurrent.ConcurrentHashMap;
import org.apache.commons.io.FileUtils;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FileSystem;
@@ -29,13 +33,31 @@ import org.apache.hadoop.fs.Path;
public class FetchFileUtils {
+ /**
+ * Per-destination lock map used to serialize concurrent symlink creation
for the same keytab
+ * file. Keyed by the normalized absolute destination path string to avoid
races caused by
+ * different path spellings referring to the same file. Entries are removed
when the corresponding
+ * {@link KerberosClient} is closed, so the map size is bounded by the
number of live catalogs.
+ */
+ private static final ConcurrentHashMap<String, Object> SYMLINK_LOCKS = new
ConcurrentHashMap<>();
+
private FetchFileUtils() {}
+ /**
+ * Removes the per-destination lock entry for the given file. Should be
called when the keytab
+ * file is deleted (e.g., on {@link KerberosClient#close()}) to prevent
unbounded map growth.
+ *
+ * @param destFile the keytab destination file whose lock entry should be
removed
+ */
+ static void removeLock(File destFile) {
+
SYMLINK_LOCKS.remove(destFile.toPath().toAbsolutePath().normalize().toString());
+ }
+
public static void fetchFileFromUri(
- String fileUri, File destFile, int timeout, Configuration conf) throws
java.io.IOException {
+ String fileUri, File destFile, int timeout, Configuration conf) throws
IOException {
try {
URI uri = new URI(fileUri);
- String scheme =
java.util.Optional.ofNullable(uri.getScheme()).orElse("file");
+ String scheme = Optional.ofNullable(uri.getScheme()).orElse("file");
switch (scheme) {
case "http":
@@ -45,7 +67,24 @@ public class FetchFileUtils {
break;
case "file":
- Files.createSymbolicLink(destFile.toPath(), new
File(uri.getPath()).toPath());
+ var srcPath = new File(uri.getPath()).toPath().normalize();
+ var destPath = destFile.toPath().toAbsolutePath().normalize();
+ Object lock = SYMLINK_LOCKS.computeIfAbsent(destPath.toString(), k
-> new Object());
+ synchronized (lock) {
+ // Skip if the symlink already points to the correct target.
+ if (Files.isSymbolicLink(destPath)
+ &&
Files.readSymbolicLink(destPath).normalize().equals(srcPath)) {
+ break;
+ }
+ // Replace via a temporary symlink + rename to minimize the window
where the
+ // keytab path is absent (which could cause loginUserFromKeytab to
fail).
+ // REPLACE_EXISTING is used here; on common local filesystems
(ext4, xfs, APFS)
+ // a same-directory rename is effectively atomic at the OS level.
+ var tmpPath = destPath.resolveSibling(destPath.getFileName() +
".symlink.tmp");
+ Files.deleteIfExists(tmpPath);
+ Files.createSymbolicLink(tmpPath, srcPath);
+ Files.move(tmpPath, destPath, StandardCopyOption.REPLACE_EXISTING);
+ }
break;
case "hdfs":
diff --git
a/catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/KerberosClient.java
b/catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/KerberosClient.java
index d1be53a115..cb115048e4 100644
---
a/catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/KerberosClient.java
+++
b/catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/KerberosClient.java
@@ -172,10 +172,6 @@ public class KerberosClient implements java.io.Closeable {
keytabsDir.mkdir();
}
File keytabFile = new File(path);
- if (keytabFile.exists() && !keytabFile.delete()) {
- throw new IllegalStateException(
- String.format("Fail to delete keytab file %s",
keytabFile.getAbsolutePath()));
- }
int fetchKeytabFileTimeout = kerberosConfig.getFetchTimeoutSec();
FetchFileUtils.fetchFileFromUri(keyTabUri, keytabFile,
fetchKeytabFileTimeout, hadoopConf);
return keytabFile;
@@ -193,6 +189,7 @@ public class KerberosClient implements java.io.Closeable {
}
Files.deleteIfExists(Paths.get(keytabFilePath));
+ FetchFileUtils.removeLock(new File(keytabFilePath));
} catch (IOException e) {
LOG.warn("Failed to delete keytab file: {}", keytabFilePath, e);
}
diff --git
a/catalogs/hive-metastore-common/src/test/java/org/apache/gravitino/hive/kerberos/TestFetchFileUtils.java
b/catalogs/hive-metastore-common/src/test/java/org/apache/gravitino/hive/kerberos/TestFetchFileUtils.java
new file mode 100644
index 0000000000..c9a4bbec28
--- /dev/null
+++
b/catalogs/hive-metastore-common/src/test/java/org/apache/gravitino/hive/kerberos/TestFetchFileUtils.java
@@ -0,0 +1,119 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.gravitino.hive.kerberos;
+
+import java.io.File;
+import java.nio.file.Files;
+import java.util.ArrayList;
+import java.util.List;
+import java.util.concurrent.CyclicBarrier;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+import java.util.concurrent.Future;
+import java.util.concurrent.TimeUnit;
+import org.apache.hadoop.conf.Configuration;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.Test;
+import org.junit.jupiter.api.io.TempDir;
+
+public class TestFetchFileUtils {
+
+ @TempDir File tempDir;
+
+ @Test
+ public void testLinkLocalFile() throws Exception {
+ File srcFile = new File(tempDir, "source");
+ Assertions.assertTrue(srcFile.createNewFile());
+ File destFile = new File(tempDir, "dest");
+
+ FetchFileUtils.fetchFileFromUri(srcFile.toURI().toString(), destFile, 10,
new Configuration());
+ Assertions.assertTrue(Files.isSymbolicLink(destFile.toPath()));
+ Assertions.assertEquals(srcFile.toPath(),
Files.readSymbolicLink(destFile.toPath()));
+ }
+
+ @Test
+ public void testConcurrentSymlinkCreation() throws Exception {
+ File srcFile = new File(tempDir, "source_concurrent");
+ Assertions.assertTrue(srcFile.createNewFile());
+ File destFile = new File(tempDir, "dest_concurrent");
+
+ int threadCount = 10;
+ CyclicBarrier barrier = new CyclicBarrier(threadCount);
+ ExecutorService executor = Executors.newFixedThreadPool(threadCount);
+ List<Future<?>> futures = new ArrayList<>();
+ try {
+ for (int i = 0; i < threadCount; i++) {
+ futures.add(
+ executor.submit(
+ () -> {
+ try {
+ barrier.await(30, TimeUnit.SECONDS);
+ FetchFileUtils.fetchFileFromUri(
+ srcFile.toURI().toString(), destFile, 10, new
Configuration());
+ } catch (Exception e) {
+ throw new RuntimeException(e);
+ }
+ }));
+ }
+ for (Future<?> future : futures) {
+ future.get(30, TimeUnit.SECONDS);
+ }
+ } finally {
+ executor.shutdownNow();
+ }
+
+ Assertions.assertTrue(Files.isSymbolicLink(destFile.toPath()));
+ Assertions.assertEquals(srcFile.toPath(),
Files.readSymbolicLink(destFile.toPath()));
+ }
+
+ @Test
+ public void testIdempotentSymlinkCreation() throws Exception {
+ File srcFile = new File(tempDir, "source_idempotent");
+ Assertions.assertTrue(srcFile.createNewFile());
+ File destFile = new File(tempDir, "dest_idempotent");
+
+ Configuration conf = new Configuration();
+ String uri = srcFile.toURI().toString();
+
+ FetchFileUtils.fetchFileFromUri(uri, destFile, 10, conf);
+ Assertions.assertTrue(Files.isSymbolicLink(destFile.toPath()));
+
+ // Second call to the same dest should succeed without error
+ FetchFileUtils.fetchFileFromUri(uri, destFile, 10, conf);
+ Assertions.assertTrue(Files.isSymbolicLink(destFile.toPath()));
+ Assertions.assertEquals(srcFile.toPath(),
Files.readSymbolicLink(destFile.toPath()));
+ }
+
+ @Test
+ public void testSymlinkReplacedWithDifferentTarget() throws Exception {
+ File srcFileA = new File(tempDir, "source_a");
+ File srcFileB = new File(tempDir, "source_b");
+ Assertions.assertTrue(srcFileA.createNewFile());
+ Assertions.assertTrue(srcFileB.createNewFile());
+ File destFile = new File(tempDir, "dest_replace");
+
+ Configuration conf = new Configuration();
+
+ FetchFileUtils.fetchFileFromUri(srcFileA.toURI().toString(), destFile, 10,
conf);
+ Assertions.assertEquals(srcFileA.toPath(),
Files.readSymbolicLink(destFile.toPath()));
+
+ FetchFileUtils.fetchFileFromUri(srcFileB.toURI().toString(), destFile, 10,
conf);
+ Assertions.assertEquals(srcFileB.toPath(),
Files.readSymbolicLink(destFile.toPath()));
+ }
+}