Copilot commented on code in PR #10742:
URL: https://github.com/apache/gravitino/pull/10742#discussion_r3071331241


##########
catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/KerberosClient.java:
##########
@@ -172,10 +172,7 @@ public File saveKeyTabFileFromUri(String path) throws 
IOException {
       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()));
-    }
+    Files.deleteIfExists(keytabFile.toPath());

Review Comment:
   `saveKeyTabFileFromUri` still deletes the destination keytab path before 
calling `FetchFileUtils.fetchFileFromUri`. For `file:` URIs this undermines the 
new “no missing-file window” behavior (and the idempotent fast-path), because 
another thread can observe the keytab path missing between `deleteIfExists` and 
the subsequent atomic replace. Consider removing this pre-delete for `file:` 
URIs (or moving the delete/replacement under the same per-destination lock / 
temp+move strategy) so concurrent Kerberos logins sharing the same keytabPath 
can’t intermittently fail with FileNotFound.
   ```suggestion
   
   ```



##########
catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/FetchFileUtils.java:
##########
@@ -45,7 +56,22 @@ public static void fetchFileFromUri(
           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;
+            }
+            // Atomically replace via a temporary symlink + rename to avoid a 
window
+            // where the keytab path is absent (which could cause 
loginUserFromKeytab to fail).
+            var tmpPath = destPath.resolveSibling(destPath.getFileName() + 
".symlink.tmp");
+            Files.deleteIfExists(tmpPath);
+            Files.createSymbolicLink(tmpPath, srcPath);
+            Files.move(tmpPath, destPath, StandardCopyOption.REPLACE_EXISTING);

Review Comment:
   The code comment says the temp-symlink + rename is “atomic”, but 
`Files.move(tmpPath, destPath, REPLACE_EXISTING)` is not guaranteed to be 
atomic unless `ATOMIC_MOVE` is used (and supported). Either add 
`StandardCopyOption.ATOMIC_MOVE` with a safe fallback, or adjust the comment to 
avoid claiming atomicity in environments where the FS/provider may implement 
move as non-atomic.



##########
catalogs/hive-metastore-common/src/test/java/org/apache/gravitino/hive/kerberos/TestFetchFileUtils.java:
##########
@@ -0,0 +1,118 @@
+/*
+ * 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 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();
+                    FetchFileUtils.fetchFileFromUri(
+                        srcFile.toURI().toString(), destFile, 10, new 
Configuration());

Review Comment:
   In the concurrency test, `barrier.await()` has no timeout. If a worker 
thread fails before reaching the barrier (or the test deadlocks), the test can 
hang indefinitely. Prefer `barrier.await(timeout, unit)` and fail the test on 
timeout so CI doesn’t get stuck.



##########
catalogs/hive-metastore-common/src/test/java/org/apache/gravitino/hive/kerberos/TestFetchFileUtils.java:
##########
@@ -0,0 +1,118 @@
+/*
+ * 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 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();
+                    FetchFileUtils.fetchFileFromUri(
+                        srcFile.toURI().toString(), destFile, 10, new 
Configuration());
+                  } catch (Exception e) {
+                    throw new RuntimeException(e);
+                  }
+                }));
+      }
+      for (Future<?> future : futures) {
+        future.get();
+      }

Review Comment:
   `future.get()` is called without a timeout. If the code under test deadlocks 
or a task never completes, this test will block indefinitely. Prefer 
`future.get(timeout, unit)` (and make the timeout generous) to keep the unit 
test suite bounded.



##########
catalogs/hive-metastore-common/src/main/java/org/apache/gravitino/hive/kerberos/FetchFileUtils.java:
##########
@@ -19,23 +19,34 @@
 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;
 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.
+   */
+  private static final ConcurrentHashMap<String, Object> SYMLINK_LOCKS = new 
ConcurrentHashMap<>();
+

Review Comment:
   `SYMLINK_LOCKS` is a static `ConcurrentHashMap` that never evicts entries. 
On a long-lived server that creates many distinct keytab destination paths 
(e.g., per-catalog/per-identity), this can grow without bound. Consider using a 
bounded/striped locking approach (e.g., Guava `Striped<Lock>`) or implementing 
safe cleanup (ref-counting or removing the entry when no longer needed).



-- 
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