Skip to content

Commit 6a509f8

Browse files
committed
HDFS-17746. [ARR] The behavior of throwing exception in getListing should be consistent with sync mode.
1 parent 5054b16 commit 6a509f8

File tree

2 files changed

+124
-13
lines changed

2 files changed

+124
-13
lines changed

hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/router/async/RouterAsyncClientProtocol.java

+12-13
Original file line numberDiff line numberDiff line change
@@ -519,25 +519,24 @@ public DirectoryListing getListing(
519519
@Override
520520
protected List<RemoteResult<RemoteLocation, DirectoryListing>> getListingInt(
521521
String src, byte[] startAfter, boolean needLocation) throws IOException {
522-
List<RemoteLocation> locations =
523-
rpcServer.getLocationsForPath(src, false, false);
524-
// Locate the dir and fetch the listing.
525-
if (locations.isEmpty()) {
526-
asyncComplete(new ArrayList<>());
527-
return asyncReturn(List.class);
528-
}
529-
asyncTry(() -> {
522+
try {
523+
List<RemoteLocation> locations =
524+
rpcServer.getLocationsForPath(src, false, false);
525+
// Locate the dir and fetch the listing.
526+
if (locations.isEmpty()) {
527+
asyncComplete(new ArrayList<>());
528+
return asyncReturn(List.class);
529+
}
530530
RemoteMethod method = new RemoteMethod("getListing",
531531
new Class<?>[] {String.class, startAfter.getClass(), boolean.class},
532532
new RemoteParam(), startAfter, needLocation);
533533
rpcClient.invokeConcurrent(locations, method, false, -1,
534534
DirectoryListing.class);
535-
});
536-
asyncCatch((CatchFunction<List, RouterResolveException>) (o, e) -> {
535+
} catch (NoLocationException | RouterResolveException e) {
537536
LOG.debug("Cannot get locations for {}, {}.", src, e.getMessage());
538-
LOG.info("Cannot get locations for {}, {}.", src, e.getMessage());
539-
return new ArrayList<>();
540-
}, RouterResolveException.class);
537+
asyncComplete(new ArrayList<>());
538+
}
539+
541540
return asyncReturn(List.class);
542541
}
543542

Original file line numberDiff line numberDiff line change
@@ -0,0 +1,112 @@
1+
/**
2+
* Licensed to the Apache Software Foundation (ASF) under one
3+
* or more contributor license agreements. See the NOTICE file
4+
* distributed with this work for additional information
5+
* regarding copyright ownership. The ASF licenses this file
6+
* to you under the Apache License, Version 2.0 (the
7+
* "License"); you may not use this file except in compliance
8+
* with the License. You may obtain a copy of the License at
9+
*
10+
* http://www.apache.org/licenses/LICENSE-2.0
11+
*
12+
* Unless required by applicable law or agreed to in writing, software
13+
* distributed under the License is distributed on an "AS IS" BASIS,
14+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15+
* See the License for the specific language governing permissions and
16+
* limitations under the License.
17+
*/
18+
package org.apache.hadoop.hdfs.server.federation.router.async;
19+
20+
import org.apache.hadoop.conf.Configuration;
21+
import org.apache.hadoop.fs.FileSystem;
22+
import org.apache.hadoop.fs.Path;
23+
import org.apache.hadoop.hdfs.server.federation.MiniRouterDFSCluster;
24+
import org.apache.hadoop.hdfs.server.federation.RouterConfigBuilder;
25+
import org.apache.hadoop.hdfs.server.federation.StateStoreDFSCluster;
26+
import org.apache.hadoop.hdfs.server.federation.resolver.MountTableManager;
27+
import org.apache.hadoop.hdfs.server.federation.resolver.MountTableResolver;
28+
import org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys;
29+
import org.apache.hadoop.hdfs.server.federation.router.Router;
30+
import org.apache.hadoop.hdfs.server.federation.router.RouterClient;
31+
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetMountTableEntriesRequest;
32+
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetMountTableEntriesResponse;
33+
import org.apache.hadoop.hdfs.server.federation.store.protocol.RemoveMountTableEntryRequest;
34+
import org.apache.hadoop.hdfs.server.federation.store.records.MountTable;
35+
import org.apache.hadoop.test.LambdaTestUtils;
36+
import org.junit.After;
37+
import org.junit.AfterClass;
38+
import org.junit.BeforeClass;
39+
import org.junit.Test;
40+
import org.slf4j.Logger;
41+
import org.slf4j.LoggerFactory;
42+
43+
import java.io.FileNotFoundException;
44+
import java.io.IOException;
45+
46+
/**
47+
* Test a router end-to-end including the MountTable using async rpc.
48+
*/
49+
public class TestRouterAsyncMountTable {
50+
public static final Logger LOG = LoggerFactory.getLogger(TestRouterAsyncMountTable.class);
51+
52+
private static StateStoreDFSCluster cluster;
53+
private static MiniRouterDFSCluster.RouterContext routerContext;
54+
private static MountTableResolver mountTable;
55+
private static FileSystem routerFs;
56+
57+
@BeforeClass
58+
public static void globalSetUp() throws Exception {
59+
// Build and start a federated cluster.
60+
cluster = new StateStoreDFSCluster(false, 2);
61+
Configuration conf = new RouterConfigBuilder()
62+
.stateStore()
63+
.admin()
64+
.rpc()
65+
.build();
66+
conf.setInt(RBFConfigKeys.DFS_ROUTER_ADMIN_MAX_COMPONENT_LENGTH_KEY, 20);
67+
conf.setBoolean(RBFConfigKeys.DFS_ROUTER_ASYNC_RPC_ENABLE_KEY, true);
68+
cluster.addRouterOverrides(conf);
69+
cluster.startCluster();
70+
cluster.startRouters();
71+
cluster.waitClusterUp();
72+
73+
routerContext = cluster.getRandomRouter();
74+
routerFs = routerContext.getFileSystem();
75+
Router router = routerContext.getRouter();
76+
mountTable = (MountTableResolver) router.getSubclusterResolver();
77+
}
78+
79+
@AfterClass
80+
public static void tearDown() {
81+
if (cluster != null) {
82+
cluster.stopRouter(routerContext);
83+
cluster.shutdown();
84+
cluster = null;
85+
}
86+
}
87+
88+
@After
89+
public void clearMountTable() throws IOException {
90+
RouterClient client = routerContext.getAdminClient();
91+
MountTableManager mountTableManager = client.getMountTableManager();
92+
GetMountTableEntriesRequest req1 =
93+
GetMountTableEntriesRequest.newInstance("/");
94+
GetMountTableEntriesResponse response =
95+
mountTableManager.getMountTableEntries(req1);
96+
for (MountTable entry : response.getEntries()) {
97+
RemoveMountTableEntryRequest req2 =
98+
RemoveMountTableEntryRequest.newInstance(entry.getSourcePath());
99+
mountTableManager.removeMountTableEntry(req2);
100+
}
101+
mountTable.setDefaultNSEnable(true);
102+
}
103+
104+
@Test
105+
public void testListNonExistPath() throws Exception {
106+
mountTable.setDefaultNSEnable(false);
107+
LambdaTestUtils.intercept(FileNotFoundException.class,
108+
"File /base does not exist.",
109+
"Expect FileNotFoundException.",
110+
() -> routerFs.listStatus(new Path("/base")));
111+
}
112+
}

0 commit comments

Comments
 (0)