Skip to content

Commit 9f85789

Browse files
author
zengqiang.xu
committed
HDFS-16724. RBF should support get the information about ancestor mount points
1 parent d8d3325 commit 9f85789

File tree

4 files changed

+207
-14
lines changed

4 files changed

+207
-14
lines changed
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,32 @@
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;
19+
20+
import java.io.IOException;
21+
22+
/**
23+
* Exception when no location found.
24+
*/
25+
public class NoLocationException extends IOException {
26+
27+
private static final long serialVersionUID = 1L;
28+
29+
public NoLocationException(String message) {
30+
super(message);
31+
}
32+
}

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

Lines changed: 20 additions & 13 deletions
Original file line numberDiff line numberDiff line change
@@ -935,19 +935,22 @@ public BatchedDirectoryListing getBatchedListing(String[] srcs,
935935
public HdfsFileStatus getFileInfo(String src) throws IOException {
936936
rpcServer.checkOperation(NameNode.OperationCategory.READ);
937937

938-
final List<RemoteLocation> locations =
939-
rpcServer.getLocationsForPath(src, false, false);
940-
RemoteMethod method = new RemoteMethod("getFileInfo",
941-
new Class<?>[] {String.class}, new RemoteParam());
942-
943938
HdfsFileStatus ret = null;
944-
// If it's a directory, we check in all locations
945-
if (rpcServer.isPathAll(src)) {
946-
ret = getFileInfoAll(locations, method);
947-
} else {
948-
// Check for file information sequentially
949-
ret = rpcClient.invokeSequential(
950-
locations, method, HdfsFileStatus.class, null);
939+
IOException noLocationException = null;
940+
try {
941+
final List<RemoteLocation> locations = rpcServer.getLocationsForPath(src, false, false);
942+
RemoteMethod method = new RemoteMethod("getFileInfo",
943+
new Class<?>[] {String.class}, new RemoteParam());
944+
945+
// If it's a directory, we check in all locations
946+
if (rpcServer.isPathAll(src)) {
947+
ret = getFileInfoAll(locations, method);
948+
} else {
949+
// Check for file information sequentially
950+
ret = rpcClient.invokeSequential(locations, method, HdfsFileStatus.class, null);
951+
}
952+
} catch (NoLocationException | RouterResolveException e) {
953+
noLocationException = e;
951954
}
952955

953956
// If there is no real path, check mount points
@@ -956,7 +959,7 @@ public HdfsFileStatus getFileInfo(String src) throws IOException {
956959
if (children != null && !children.isEmpty()) {
957960
Map<String, Long> dates = getMountPointDates(src);
958961
long date = 0;
959-
if (dates != null && dates.containsKey(src)) {
962+
if (dates.containsKey(src)) {
960963
date = dates.get(src);
961964
}
962965
ret = getMountPointStatus(src, children.size(), date);
@@ -966,6 +969,10 @@ public HdfsFileStatus getFileInfo(String src) throws IOException {
966969
}
967970
}
968971

972+
if (ret == null && noLocationException != null) {
973+
throw noLocationException;
974+
}
975+
969976
return ret;
970977
}
971978

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

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -1731,7 +1731,7 @@ protected List<RemoteLocation> getLocationsForPath(String path,
17311731
final PathLocation location =
17321732
this.subclusterResolver.getDestinationForPath(path);
17331733
if (location == null) {
1734-
throw new IOException("Cannot find locations for " + path + " in " +
1734+
throw new NoLocationException("Cannot find locations for " + path + " in " +
17351735
this.subclusterResolver.getClass().getSimpleName());
17361736
}
17371737

Original file line numberDiff line numberDiff line change
@@ -0,0 +1,154 @@
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;
19+
20+
import org.apache.hadoop.conf.Configuration;
21+
import org.apache.hadoop.hdfs.protocol.ClientProtocol;
22+
import org.apache.hadoop.hdfs.protocol.HdfsFileStatus;
23+
import org.apache.hadoop.hdfs.server.federation.MiniRouterDFSCluster.RouterContext;
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.store.protocol.AddMountTableEntryRequest;
29+
import org.apache.hadoop.hdfs.server.federation.store.protocol.AddMountTableEntryResponse;
30+
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetMountTableEntriesRequest;
31+
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetMountTableEntriesResponse;
32+
import org.apache.hadoop.hdfs.server.federation.store.protocol.RemoveMountTableEntryRequest;
33+
import org.apache.hadoop.hdfs.server.federation.store.records.MountTable;
34+
import org.junit.After;
35+
import org.junit.AfterClass;
36+
import org.junit.BeforeClass;
37+
import org.junit.Test;
38+
39+
import java.io.IOException;
40+
import java.util.Collections;
41+
42+
import static org.junit.Assert.assertEquals;
43+
import static org.junit.Assert.assertNotNull;
44+
import static org.junit.Assert.assertTrue;
45+
import static org.junit.Assert.fail;
46+
47+
/**
48+
* Test a router end-to-end including the MountTable without default nameservice.
49+
*/
50+
public class TestRouterMountTableWithoutDefaultNS {
51+
private static StateStoreDFSCluster cluster;
52+
private static RouterContext routerContext;
53+
private static MountTableResolver mountTable;
54+
private static ClientProtocol routerProtocol;
55+
56+
@BeforeClass
57+
public static void globalSetUp() throws Exception {
58+
// Build and start a federated cluster
59+
cluster = new StateStoreDFSCluster(false, 2);
60+
Configuration conf = new RouterConfigBuilder()
61+
.stateStore()
62+
.admin()
63+
.rpc()
64+
.build();
65+
conf.setInt(RBFConfigKeys.DFS_ROUTER_ADMIN_MAX_COMPONENT_LENGTH_KEY, 20);
66+
conf.setBoolean(RBFConfigKeys.DFS_ROUTER_DEFAULT_NAMESERVICE_ENABLE, false);
67+
cluster.addRouterOverrides(conf);
68+
cluster.startCluster();
69+
cluster.startRouters();
70+
cluster.waitClusterUp();
71+
72+
// Get the end points
73+
routerContext = cluster.getRandomRouter();
74+
Router router = routerContext.getRouter();
75+
routerProtocol = routerContext.getClient().getNamenode();
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+
}
102+
103+
/**
104+
* Add a mount table entry to the mount table through the admin API.
105+
* @param entry Mount table entry to add.
106+
* @return If it was succesfully added.
107+
* @throws IOException Problems adding entries.
108+
*/
109+
private boolean addMountTable(final MountTable entry) throws IOException {
110+
RouterClient client = routerContext.getAdminClient();
111+
MountTableManager mountTableManager = client.getMountTableManager();
112+
AddMountTableEntryRequest addRequest =
113+
AddMountTableEntryRequest.newInstance(entry);
114+
AddMountTableEntryResponse addResponse =
115+
mountTableManager.addMountTableEntry(addRequest);
116+
117+
// Reload the Router cache
118+
mountTable.loadCache(true);
119+
120+
return addResponse.getStatus();
121+
}
122+
123+
/**
124+
* Verify that RBF that disable default nameservice should support
125+
* get information about ancestor mount points.
126+
*/
127+
@Test
128+
public void testGetFileInfoWithSubMountPoint() throws IOException {
129+
MountTable addEntry = MountTable.newInstance("/testdir/1",
130+
Collections.singletonMap("ns0", "/testdir/1"));
131+
assertTrue(addMountTable(addEntry));
132+
HdfsFileStatus finfo = routerProtocol.getFileInfo("/testdir");
133+
assertNotNull(finfo);
134+
assertEquals("supergroup", finfo.getGroup());
135+
assertTrue(finfo.isDirectory());
136+
}
137+
138+
/**
139+
* Verify that RBF doesn't support get the file information
140+
* with no location and sub mount points.
141+
*/
142+
@Test
143+
public void testGetFileInfoWithoutSubMountPoint() throws IOException {
144+
MountTable addEntry = MountTable.newInstance("/testdir/1",
145+
Collections.singletonMap("ns0", "/testdir/1"));
146+
assertTrue(addMountTable(addEntry));
147+
try {
148+
routerProtocol.getFileInfo("/testdir2");
149+
fail();
150+
} catch (IOException ioe) {
151+
assertTrue(ioe.getMessage().contains("Cannot find locations for /testdir2"));
152+
}
153+
}
154+
}

0 commit comments

Comments
 (0)