Selaa lähdekoodia

HDFS-17115. HttpFS Add Support getErasureCodeCodecs API (#5875). Contributed by Hualong Zhang.

Reviewed-by: Shilun Fan <slfan1989@apache.org>
Signed-off-by: Ayush Saxena <ayushsaxena@apache.org>
zhtttylz 1 vuosi sitten
vanhempi
commit
068d8c7e4d

+ 12 - 0
hadoop-hdfs-project/hadoop-hdfs-httpfs/src/main/java/org/apache/hadoop/fs/http/client/HttpFSFileSystem.java

@@ -286,6 +286,7 @@ public class HttpFSFileSystem extends FileSystem
     GETFILELINKSTATUS(HTTP_GET),
     GETSTATUS(HTTP_GET),
     GETECPOLICIES(HTTP_GET),
+    GETECCODECS(HTTP_GET),
     GET_BLOCK_LOCATIONS(HTTP_GET);
 
     private String httpMethod;
@@ -1786,6 +1787,17 @@ public class HttpFSFileSystem extends FileSystem
     return JsonUtilClient.getAllErasureCodingPolicies(json);
   }
 
+  public Map<String, String> getAllErasureCodingCodecs() throws IOException {
+    Map<String, String> params = new HashMap<>();
+    params.put(OP_PARAM, Operation.GETECCODECS.toString());
+    Path path = new Path(getUri().toString(), "/");
+    HttpURLConnection conn =
+        getConnection(Operation.GETECCODECS.getMethod(), params, path, false);
+    HttpExceptionUtils.validateResponse(conn, HttpURLConnection.HTTP_OK);
+    JSONObject json = (JSONObject) HttpFSUtils.jsonParse(conn);
+    return JsonUtilClient.getErasureCodeCodecs(json);
+  }
+
   @VisibleForTesting
   static BlockLocation[] toBlockLocations(JSONObject json) throws IOException {
     ObjectMapper mapper = new ObjectMapper();

+ 27 - 0
hadoop-hdfs-project/hadoop-hdfs-httpfs/src/main/java/org/apache/hadoop/fs/http/server/FSOperations.java

@@ -65,6 +65,7 @@ import java.io.InputStream;
 import java.io.OutputStream;
 import java.util.Collection;
 import java.util.EnumSet;
+import java.util.HashMap;
 import java.util.LinkedHashMap;
 import java.util.List;
 import java.util.Map;
@@ -2369,4 +2370,30 @@ public final class FSOperations {
       return JsonUtil.toJsonString(ecPolicyInfos.stream().toArray(ErasureCodingPolicyInfo[]::new));
     }
   }
+
+  /**
+   * Executor that performs a FSGetErasureCodingCodecs operation.
+   */
+  @InterfaceAudience.Private
+  public static class FSGetErasureCodingCodecs
+      implements FileSystemAccess.FileSystemExecutor<Map> {
+
+    public FSGetErasureCodingCodecs() {
+    }
+
+    @Override
+    public Map execute(FileSystem fs) throws IOException {
+      Map<String, Map<String, String>> ecCodecs = new HashMap<>();
+      if (fs instanceof DistributedFileSystem) {
+        DistributedFileSystem dfs = (DistributedFileSystem) fs;
+        ecCodecs.put("ErasureCodingCodecs", dfs.getAllErasureCodingCodecs());
+      } else {
+        throw new UnsupportedOperationException("getErasureCodeCodecs is " +
+            "not supported for HttpFs on " + fs.getClass() +
+            ". Please check your fs.defaultFS configuration");
+      }
+      HttpFSServerWebApp.get().getMetrics().incrOpsECCodecs();
+      return ecCodecs;
+    }
+  }
 }

+ 1 - 0
hadoop-hdfs-project/hadoop-hdfs-httpfs/src/main/java/org/apache/hadoop/fs/http/server/HttpFSParametersProvider.java

@@ -131,6 +131,7 @@ public class HttpFSParametersProvider extends ParametersProvider {
     PARAMS_DEF.put(Operation.GETFILELINKSTATUS, new Class[]{});
     PARAMS_DEF.put(Operation.GETSTATUS, new Class[]{});
     PARAMS_DEF.put(Operation.GETECPOLICIES, new Class[]{});
+    PARAMS_DEF.put(Operation.GETECCODECS, new Class[]{});
     PARAMS_DEF.put(Operation.GET_BLOCK_LOCATIONS, new Class[] {OffsetParam.class, LenParam.class});
   }
 

+ 8 - 0
hadoop-hdfs-project/hadoop-hdfs-httpfs/src/main/java/org/apache/hadoop/fs/http/server/HttpFSServer.java

@@ -534,6 +534,14 @@ public class HttpFSServer {
       response = Response.ok(js).type(MediaType.APPLICATION_JSON).build();
       break;
     }
+    case GETECCODECS: {
+      FSOperations.FSGetErasureCodingCodecs command =
+          new FSOperations.FSGetErasureCodingCodecs();
+      Map json = fsExecute(user, command);
+      AUDIT_LOG.info("[{}]", path);
+      response = Response.ok(json).type(MediaType.APPLICATION_JSON).build();
+      break;
+    }
     case GET_BLOCK_LOCATIONS: {
       long offset = 0;
       long len = Long.MAX_VALUE;

+ 5 - 0
hadoop-hdfs-project/hadoop-hdfs-httpfs/src/main/java/org/apache/hadoop/fs/http/server/metrics/HttpFSServerMetrics.java

@@ -66,6 +66,7 @@ public class HttpFSServerMetrics {
   private @Metric MutableCounterLong opsCheckAccess;
   private @Metric MutableCounterLong opsStatus;
   private @Metric MutableCounterLong opsAllECPolicies;
+  private @Metric MutableCounterLong opsECCodecs;
 
   private final MetricsRegistry registry = new MetricsRegistry("httpfsserver");
   private final String name;
@@ -170,4 +171,8 @@ public class HttpFSServerMetrics {
   public void incrOpsAllECPolicies() {
     opsAllECPolicies.incr();
   }
+
+  public void incrOpsECCodecs() {
+    opsECCodecs.incr();
+  }
 }

+ 54 - 2
hadoop-hdfs-project/hadoop-hdfs-httpfs/src/test/java/org/apache/hadoop/fs/http/client/BaseTestHttpFSWith.java

@@ -105,6 +105,7 @@ import java.util.Collection;
 import java.util.List;
 import java.util.Map;
 import java.util.regex.Pattern;
+import java.util.concurrent.atomic.AtomicReference;
 
 import static org.junit.Assert.assertArrayEquals;
 import static org.junit.Assert.assertEquals;
@@ -1218,9 +1219,9 @@ public abstract class BaseTestHttpFSWith extends HFSTestCase {
     FILE_STATUS_ATTR, GET_SNAPSHOT_DIFF, GET_SNAPSHOTTABLE_DIRECTORY_LIST,
     GET_SNAPSHOT_LIST, GET_SERVERDEFAULTS, CHECKACCESS, SETECPOLICY,
     SATISFYSTORAGEPOLICY, GET_SNAPSHOT_DIFF_LISTING, GETFILEBLOCKLOCATIONS,
-    GETFILELINKSTATUS, GETSTATUS, GETECPOLICIES
+    GETFILELINKSTATUS, GETSTATUS, GETECPOLICIES, GETECCODECS
   }
-
+  @SuppressWarnings("methodlength")
   private void operation(Operation op) throws Exception {
     switch (op) {
     case GET:
@@ -1370,6 +1371,9 @@ public abstract class BaseTestHttpFSWith extends HFSTestCase {
     case GETECPOLICIES:
       testGetAllEEPolicies();
       break;
+    case GETECCODECS:
+      testGetECCodecs();
+      break;
     }
   }
 
@@ -2149,6 +2153,54 @@ public abstract class BaseTestHttpFSWith extends HFSTestCase {
     }
   }
 
+  private void testGetECCodecs() throws Exception {
+    if (isLocalFS()) {
+      // do not test the testGetECCodecs for local FS.
+      return;
+    }
+    final Path path = new Path("/foo");
+
+    FileSystem fs = FileSystem.get(path.toUri(), this.getProxiedFSConf());
+    LambdaTestUtils.intercept(AssertionError.class, () -> {
+      if (!(fs instanceof DistributedFileSystem)) {
+        throw new AssertionError(fs.getClass().getSimpleName() +
+            " is not of type DistributedFileSystem.");
+      }
+    });
+
+    DistributedFileSystem dfs =
+        (DistributedFileSystem) FileSystem.get(path.toUri(), this.getProxiedFSConf());
+    FileSystem httpFs = this.getHttpFSFileSystem();
+
+    Map<String, String> dfsErasureCodingCodecs = dfs.getAllErasureCodingCodecs();
+
+    final AtomicReference<Map<String, String>> diffErasureCodingCodecsRef =
+        new AtomicReference<>();
+    LambdaTestUtils.intercept(AssertionError.class, () -> {
+      if (httpFs instanceof HttpFSFileSystem) {
+        HttpFSFileSystem httpFSFileSystem = (HttpFSFileSystem) httpFs;
+        diffErasureCodingCodecsRef.set(httpFSFileSystem.getAllErasureCodingCodecs());
+      } else if (httpFs instanceof WebHdfsFileSystem) {
+        WebHdfsFileSystem webHdfsFileSystem = (WebHdfsFileSystem) httpFs;
+        diffErasureCodingCodecsRef.set(webHdfsFileSystem.getAllErasureCodingCodecs());
+      } else {
+        throw new AssertionError(httpFs.getClass().getSimpleName() +
+            " is not of type HttpFSFileSystem or WebHdfsFileSystem");
+      }
+    });
+    Map<String, String> diffErasureCodingCodecs = diffErasureCodingCodecsRef.get();
+
+    //Validate testGetECCodecs are the same as DistributedFileSystem
+    Assert.assertEquals(dfsErasureCodingCodecs.size(), diffErasureCodingCodecs.size());
+
+    for (Map.Entry<String, String> entry : dfsErasureCodingCodecs.entrySet()) {
+      String key = entry.getKey();
+      String value = entry.getValue();
+      Assert.assertTrue(diffErasureCodingCodecs.containsKey(key));
+      Assert.assertEquals(value, diffErasureCodingCodecs.get(key));
+    }
+  }
+
   private void assertHttpFsReportListingWithDfsClient(SnapshotDiffReportListing diffReportListing,
       SnapshotDiffReportListing dfsDiffReportListing) {
     Assert.assertEquals(diffReportListing.getCreateList().size(),