Skip to content

Commit

Permalink
HADOOP-19346. ViewFileSystem.InnerCache: Replaced ReentrantReadWriteL…
Browse files Browse the repository at this point in the history
…ock with ConcurrentHashMap/putIfAbsent() (#7187)

* HADOOP-19346 ViewFileSystem.InnerCache: Replaced ReentrantReadWriteLock with ConcurrentHashMap/putIfAbsent()
  • Loading branch information
xinglin authored Nov 27, 2024
1 parent 65a5bf3 commit 964e089
Showing 1 changed file with 20 additions and 30 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -28,6 +28,8 @@
import static org.apache.hadoop.fs.viewfs.Constants.CONFIG_VIEWFS_TRASH_FORCE_INSIDE_MOUNT_POINT;
import static org.apache.hadoop.fs.viewfs.Constants.CONFIG_VIEWFS_TRASH_FORCE_INSIDE_MOUNT_POINT_DEFAULT;

import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.ConcurrentMap;
import java.util.function.Function;
import java.io.FileNotFoundException;
import java.io.IOException;
Expand All @@ -45,7 +47,6 @@
import java.util.Map.Entry;
import java.util.Objects;
import java.util.Set;
import java.util.concurrent.locks.ReentrantReadWriteLock;

import org.apache.hadoop.util.Preconditions;
import org.apache.hadoop.classification.InterfaceAudience;
Expand Down Expand Up @@ -118,38 +119,32 @@ protected FsGetter fsGetter() {
* Caching children filesystems. HADOOP-15565.
*/
static class InnerCache {
private Map<Key, FileSystem> map = new HashMap<>();
private ConcurrentMap<Key, FileSystem> map = new ConcurrentHashMap<>();
private FsGetter fsCreator;
private ReentrantReadWriteLock rwLock = new ReentrantReadWriteLock();

InnerCache(FsGetter fsCreator) {
this.fsCreator = fsCreator;
}

FileSystem get(URI uri, Configuration config) throws IOException {
Key key = new Key(uri);
FileSystem fs = null;
// computeIfAbsent() does not support a mapping function which throws IOException.
// Wrap fsCreator.getNewInstance() to not throw IOException and return null instead.
FileSystem getNewFileSystem(URI uri, Configuration config) {
try {
rwLock.readLock().lock();
fs = map.get(key);
if (fs != null) {
return fs;
}
} finally {
rwLock.readLock().unlock();
return fsCreator.getNewInstance(uri, config);
} catch (IOException e) {
LOG.error("Failed to create new FileSystem instance for " + uri, e);
return null;
}
try {
rwLock.writeLock().lock();
fs = map.get(key);
if (fs != null) {
return fs;
}
fs = fsCreator.getNewInstance(uri, config);
map.put(key, fs);
return fs;
} finally {
rwLock.writeLock().unlock();
}

FileSystem get(URI uri, Configuration config) throws IOException {
Key key = new Key(uri);

FileSystem fs = map.computeIfAbsent(key, k -> getNewFileSystem(uri, config));
if (fs == null) {
throw new IOException("Failed to create new FileSystem instance for " + uri);
}
return fs;
}

void closeAll() {
Expand All @@ -163,12 +158,7 @@ void closeAll() {
}

void clear() {
try {
rwLock.writeLock().lock();
map.clear();
} finally {
rwLock.writeLock().unlock();
}
map.clear();
}

/**
Expand Down

0 comments on commit 964e089

Please sign in to comment.