You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@ozone.apache.org by GitBox <gi...@apache.org> on 2022/05/13 08:03:59 UTC

[GitHub] [ozone] kaijchen opened a new pull request, #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

kaijchen opened a new pull request, #3410:
URL: https://github.com/apache/ozone/pull/3410

   ## What changes were proposed in this pull request?
   
   Add listBlock in ContainerProtocolCalls.
   
   ## What is the link to the Apache JIRA
   
   https://issues.apache.org/jira/browse/HDDS-6596
   
   ## How was this patch tested?
   


-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r872165629


##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/ContainerProtocolCalls.java:
##########
@@ -73,6 +75,50 @@ public final class ContainerProtocolCalls  {
   private ContainerProtocolCalls() {
   }
 
+  /**
+   * Calls the container protocol to list blocks in container.
+   *
+   * @param xceiverClient client to perform call
+   * @param containerID the ID of the container to list block
+   * @param replicaIndex the index of the replica in pipeline
+   * @param startLocalID the localID of the first block to get
+   * @param count max number of blocks to get
+   * @param token a token for this block (may be null)
+   * @return container protocol list block response
+   * @throws IOException if there is an I/O error while performing the call
+   */
+  public static ListBlockResponseProto listBlock(XceiverClientSpi xceiverClient,
+      long containerID, int replicaIndex, Long startLocalID, int count,
+      Token<? extends TokenIdentifier> token) throws IOException {
+
+    ListBlockRequestProto.Builder listBlockBuilder =
+        ListBlockRequestProto.newBuilder()
+            .setCount(count);
+
+    if (startLocalID != null) {
+      listBlockBuilder.setStartLocalID(startLocalID);
+    }
+
+    String datanodeID = xceiverClient.getPipeline().getNodesInOrder()
+        .get(replicaIndex).getUuidString();
+
+    ContainerCommandRequestProto.Builder builder =
+        ContainerCommandRequestProto.newBuilder()
+            .setCmdType(Type.ListBlock)
+            .setContainerID(containerID)
+            .setDatanodeUuid(datanodeID)
+            .setListBlock(listBlockBuilder.build());
+
+    if (token != null) {
+      builder.setEncodedToken(token.encodeToUrlString());
+    }
+
+    ContainerCommandRequestProto request = builder.build();
+    ContainerCommandResponseProto response =
+        xceiverClient.sendCommand(request, getValidatorList());

Review Comment:
   We may need to call `xceiverClient#sendCommandOnAllNodes` here because of the `sendCommand` node picking strategy. What do you think @umamaheswararao?
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L457-L458
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L141-L142
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L154-L155



##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/ContainerProtocolCalls.java:
##########
@@ -73,6 +75,50 @@ public final class ContainerProtocolCalls  {
   private ContainerProtocolCalls() {
   }
 
+  /**
+   * Calls the container protocol to list blocks in container.
+   *
+   * @param xceiverClient client to perform call
+   * @param containerID the ID of the container to list block
+   * @param replicaIndex the index of the replica in pipeline
+   * @param startLocalID the localID of the first block to get
+   * @param count max number of blocks to get
+   * @param token a token for this block (may be null)
+   * @return container protocol list block response
+   * @throws IOException if there is an I/O error while performing the call
+   */
+  public static ListBlockResponseProto listBlock(XceiverClientSpi xceiverClient,
+      long containerID, int replicaIndex, Long startLocalID, int count,
+      Token<? extends TokenIdentifier> token) throws IOException {
+
+    ListBlockRequestProto.Builder listBlockBuilder =
+        ListBlockRequestProto.newBuilder()
+            .setCount(count);
+
+    if (startLocalID != null) {
+      listBlockBuilder.setStartLocalID(startLocalID);
+    }
+
+    String datanodeID = xceiverClient.getPipeline().getNodesInOrder()
+        .get(replicaIndex).getUuidString();
+
+    ContainerCommandRequestProto.Builder builder =
+        ContainerCommandRequestProto.newBuilder()
+            .setCmdType(Type.ListBlock)
+            .setContainerID(containerID)
+            .setDatanodeUuid(datanodeID)
+            .setListBlock(listBlockBuilder.build());
+
+    if (token != null) {
+      builder.setEncodedToken(token.encodeToUrlString());
+    }
+
+    ContainerCommandRequestProto request = builder.build();
+    ContainerCommandResponseProto response =
+        xceiverClient.sendCommand(request, getValidatorList());

Review Comment:
   We may need to call `xceiverClient#sendCommandOnAllNodes` here because of the `sendCommand` node picking strategy. What do you think @umamaheswararao?
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L457-L458
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L141-L142
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L154-L155



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r874288557


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {
+
+  private static MiniOzoneCluster cluster;
+  private static OzoneManager om;
+  private static StorageContainerManager scm;
+  private static OzoneClient client;
+  private static ObjectStore store;
+  private static StorageContainerLocationProtocolClientSideTranslatorPB
+      storageContainerLocationClient;
+  private static final String SCM_ID = UUID.randomUUID().toString();
+  private static final String CLUSTER_ID = UUID.randomUUID().toString();
+  private static final int EC_DATA = 10;
+  private static final int EC_PARITY = 4;
+  private static final EcCodec EC_CODEC = EcCodec.RS;
+  private static final int EC_CHUNK_SIZE = 1024;
+  private static final int NUM_DN = EC_DATA + EC_PARITY;
+  private static final int[] KEY_SIZES = new int[]{
+      EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * 3 / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE * 3 / 2,
+  };
+  private static byte[][] values;
+  private static long containerID;
+  private static Pipeline pipeline;
+  private static List<DatanodeDetails> datanodeDetails;
+  private List<XceiverClientSpi> clients = null;
+
+  @BeforeAll
+  public static void init() throws Exception {
+    OzoneConfiguration conf = new OzoneConfiguration();
+    conf.setInt(ScmConfigKeys.OZONE_SCM_PIPELINE_OWNER_CONTAINER_COUNT, 1);
+    conf.setBoolean(OzoneConfigKeys.OZONE_ACL_ENABLED, true);
+    conf.set(OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS,
+        OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS_NATIVE);
+    startCluster(conf);
+    prepareData(KEY_SIZES);
+  }
+
+  @AfterAll
+  public static void stop() throws IOException {
+    stopCluster();
+  }
+
+  private Pipeline createSingleNodePipeline(Pipeline ecPipeline,
+      DatanodeDetails node, int replicaIndex) {
+
+    Map<DatanodeDetails, Integer> indicesForSinglePipeline = new HashMap<>();
+    indicesForSinglePipeline.put(node, replicaIndex);
+
+    return Pipeline.newBuilder()
+        .setId(ecPipeline.getId())
+        .setReplicationConfig(ecPipeline.getReplicationConfig())
+        .setState(ecPipeline.getPipelineState())
+        .setNodes(ImmutableList.of(node))
+        .setReplicaIndexes(indicesForSinglePipeline)
+        .build();
+  }
+
+  @BeforeEach
+  public void connectToDatanodes() {
+    clients = new ArrayList<>(datanodeDetails.size());
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      clients.add(new XceiverClientGrpc(
+          createSingleNodePipeline(pipeline, datanodeDetails.get(i), i + 1),
+          cluster.getConf()));
+    }
+  }
+
+  @AfterEach
+  public void closeClients() {
+    if (clients == null) {
+      return;
+    }
+    for (XceiverClientSpi c : clients) {
+      c.close();
+    }
+    clients = null;
+  }
+
+  private Function<Integer, Integer> chunksInReplicaFunc(int i) {
+    if (i < EC_DATA) {
+      return (keySize) -> {
+        int dataBlocks = (keySize + EC_CHUNK_SIZE - 1) / EC_CHUNK_SIZE;
+        return (dataBlocks + EC_DATA - 1 - i) / EC_DATA;
+      };
+    } else {
+      return (keySize) -> (keySize + EC_CHUNK_SIZE * EC_DATA - 1) /
+          (EC_CHUNK_SIZE * EC_DATA);
+    }
+  }
+
+  @Test
+  public void testListBlock() throws Exception {
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      final int minKeySize = i < EC_DATA ? i * EC_CHUNK_SIZE : 0;
+      int numExpectedBlocks = (int) Arrays.stream(KEY_SIZES)
+          .filter(size -> size > minKeySize).count();
+      Function<Integer, Integer> expectedChunksFunc = chunksInReplicaFunc(i);
+      int numExpectedChunks = Arrays.stream(KEY_SIZES)
+          .map(expectedChunksFunc::apply).sum();
+      try {
+        ListBlockResponseProto response = ContainerProtocolCalls.listBlock(
+            clients.get(i), containerID, null, numExpectedBlocks + 1, null);
+        Assertions.assertEquals(numExpectedBlocks, response.getBlockDataCount(),
+            "blocks count doesn't match on DN " + i);
+        Assertions.assertEquals(numExpectedChunks,
+            response.getBlockDataList().stream()
+                .mapToInt(BlockData::getChunksCount).sum(),
+            "chunks count doesn't match on DN " + i);
+      } catch (StorageContainerException e) {

Review Comment:
   I have changed the code to check zero blocks expected in advance.



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r872171432


##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/ContainerProtocolCalls.java:
##########
@@ -73,6 +75,50 @@ public final class ContainerProtocolCalls  {
   private ContainerProtocolCalls() {
   }
 
+  /**
+   * Calls the container protocol to list blocks in container.
+   *
+   * @param xceiverClient client to perform call
+   * @param containerID the ID of the container to list block
+   * @param replicaIndex the index of the replica in pipeline
+   * @param startLocalID the localID of the first block to get
+   * @param count max number of blocks to get
+   * @param token a token for this block (may be null)
+   * @return container protocol list block response
+   * @throws IOException if there is an I/O error while performing the call
+   */
+  public static ListBlockResponseProto listBlock(XceiverClientSpi xceiverClient,
+      long containerID, int replicaIndex, Long startLocalID, int count,
+      Token<? extends TokenIdentifier> token) throws IOException {
+
+    ListBlockRequestProto.Builder listBlockBuilder =
+        ListBlockRequestProto.newBuilder()
+            .setCount(count);
+
+    if (startLocalID != null) {
+      listBlockBuilder.setStartLocalID(startLocalID);
+    }
+
+    String datanodeID = xceiverClient.getPipeline().getNodesInOrder()
+        .get(replicaIndex).getUuidString();
+
+    ContainerCommandRequestProto.Builder builder =
+        ContainerCommandRequestProto.newBuilder()
+            .setCmdType(Type.ListBlock)
+            .setContainerID(containerID)
+            .setDatanodeUuid(datanodeID)

Review Comment:
   Seems setting `DatanodeUUID` here is useless, this is the only place where it is read.
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/impl/HddsDispatcher.java#L423



##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/ContainerProtocolCalls.java:
##########
@@ -73,6 +75,50 @@ public final class ContainerProtocolCalls  {
   private ContainerProtocolCalls() {
   }
 
+  /**
+   * Calls the container protocol to list blocks in container.
+   *
+   * @param xceiverClient client to perform call
+   * @param containerID the ID of the container to list block
+   * @param replicaIndex the index of the replica in pipeline
+   * @param startLocalID the localID of the first block to get
+   * @param count max number of blocks to get
+   * @param token a token for this block (may be null)
+   * @return container protocol list block response
+   * @throws IOException if there is an I/O error while performing the call
+   */
+  public static ListBlockResponseProto listBlock(XceiverClientSpi xceiverClient,
+      long containerID, int replicaIndex, Long startLocalID, int count,
+      Token<? extends TokenIdentifier> token) throws IOException {
+
+    ListBlockRequestProto.Builder listBlockBuilder =
+        ListBlockRequestProto.newBuilder()
+            .setCount(count);
+
+    if (startLocalID != null) {
+      listBlockBuilder.setStartLocalID(startLocalID);
+    }
+
+    String datanodeID = xceiverClient.getPipeline().getNodesInOrder()
+        .get(replicaIndex).getUuidString();
+
+    ContainerCommandRequestProto.Builder builder =
+        ContainerCommandRequestProto.newBuilder()
+            .setCmdType(Type.ListBlock)
+            .setContainerID(containerID)
+            .setDatanodeUuid(datanodeID)

Review Comment:
   Seems setting `DatanodeUUID` here is useless, this is the only place where it is read.
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/impl/HddsDispatcher.java#L423



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on PR #3410:
URL: https://github.com/apache/ozone/pull/3410#issuecomment-1127318334

   Hi @umamaheswararao, in my previous attempt, I added 2 new integration tests to verify the `ListBlock` in
   https://github.com/kaijchen/ozone/tree/offline-recovery/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/replication
   
   Do you think we need tests like these in this PR? And where should I put them?
   
   Thanks.


-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] umamaheswararao commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
umamaheswararao commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r875443020


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {

Review Comment:
   Make sense. Thanks for addressing this.



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r875412083


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {

Review Comment:
   Thanks for the suggestion, I have moved this test to `org.apache.hadoop.hdds.scm.storage` since we are testing `ContainerProtocolCalls`. And I have renamed the class to `TestContainerCommandsEC` to reflect that we are testing them on EC containers.



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r874273537


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {
+
+  private static MiniOzoneCluster cluster;
+  private static OzoneManager om;
+  private static StorageContainerManager scm;
+  private static OzoneClient client;
+  private static ObjectStore store;
+  private static StorageContainerLocationProtocolClientSideTranslatorPB
+      storageContainerLocationClient;
+  private static final String SCM_ID = UUID.randomUUID().toString();
+  private static final String CLUSTER_ID = UUID.randomUUID().toString();
+  private static final int EC_DATA = 10;
+  private static final int EC_PARITY = 4;
+  private static final EcCodec EC_CODEC = EcCodec.RS;
+  private static final int EC_CHUNK_SIZE = 1024;
+  private static final int NUM_DN = EC_DATA + EC_PARITY;
+  private static final int[] KEY_SIZES = new int[]{
+      EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * 3 / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE * 3 / 2,
+  };
+  private static byte[][] values;
+  private static long containerID;
+  private static Pipeline pipeline;
+  private static List<DatanodeDetails> datanodeDetails;
+  private List<XceiverClientSpi> clients = null;
+
+  @BeforeAll
+  public static void init() throws Exception {
+    OzoneConfiguration conf = new OzoneConfiguration();
+    conf.setInt(ScmConfigKeys.OZONE_SCM_PIPELINE_OWNER_CONTAINER_COUNT, 1);
+    conf.setBoolean(OzoneConfigKeys.OZONE_ACL_ENABLED, true);
+    conf.set(OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS,
+        OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS_NATIVE);
+    startCluster(conf);
+    prepareData(KEY_SIZES);
+  }
+
+  @AfterAll
+  public static void stop() throws IOException {
+    stopCluster();
+  }
+
+  private Pipeline createSingleNodePipeline(Pipeline ecPipeline,
+      DatanodeDetails node, int replicaIndex) {
+
+    Map<DatanodeDetails, Integer> indicesForSinglePipeline = new HashMap<>();
+    indicesForSinglePipeline.put(node, replicaIndex);
+
+    return Pipeline.newBuilder()
+        .setId(ecPipeline.getId())
+        .setReplicationConfig(ecPipeline.getReplicationConfig())
+        .setState(ecPipeline.getPipelineState())
+        .setNodes(ImmutableList.of(node))
+        .setReplicaIndexes(indicesForSinglePipeline)
+        .build();
+  }
+
+  @BeforeEach
+  public void connectToDatanodes() {
+    clients = new ArrayList<>(datanodeDetails.size());
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      clients.add(new XceiverClientGrpc(
+          createSingleNodePipeline(pipeline, datanodeDetails.get(i), i + 1),
+          cluster.getConf()));
+    }
+  }
+
+  @AfterEach
+  public void closeClients() {
+    if (clients == null) {
+      return;
+    }
+    for (XceiverClientSpi c : clients) {
+      c.close();
+    }
+    clients = null;
+  }
+
+  private Function<Integer, Integer> chunksInReplicaFunc(int i) {
+    if (i < EC_DATA) {
+      return (keySize) -> {
+        int dataBlocks = (keySize + EC_CHUNK_SIZE - 1) / EC_CHUNK_SIZE;
+        return (dataBlocks + EC_DATA - 1 - i) / EC_DATA;
+      };
+    } else {
+      return (keySize) -> (keySize + EC_CHUNK_SIZE * EC_DATA - 1) /
+          (EC_CHUNK_SIZE * EC_DATA);
+    }
+  }
+
+  @Test
+  public void testListBlock() throws Exception {
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      final int minKeySize = i < EC_DATA ? i * EC_CHUNK_SIZE : 0;
+      int numExpectedBlocks = (int) Arrays.stream(KEY_SIZES)
+          .filter(size -> size > minKeySize).count();
+      Function<Integer, Integer> expectedChunksFunc = chunksInReplicaFunc(i);
+      int numExpectedChunks = Arrays.stream(KEY_SIZES)
+          .map(expectedChunksFunc::apply).sum();
+      try {
+        ListBlockResponseProto response = ContainerProtocolCalls.listBlock(
+            clients.get(i), containerID, null, numExpectedBlocks + 1, null);
+        Assertions.assertEquals(numExpectedBlocks, response.getBlockDataCount(),
+            "blocks count doesn't match on DN " + i);
+        Assertions.assertEquals(numExpectedChunks,
+            response.getBlockDataList().stream()
+                .mapToInt(BlockData::getChunksCount).sum(),
+            "chunks count doesn't match on DN " + i);
+      } catch (StorageContainerException e) {

Review Comment:
   If we only write one small key which only takes 1 data cell, the rest data cells will be empty.
   And we will get container not found exception from these datanodes.



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] umamaheswararao commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
umamaheswararao commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r872936634


##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/ContainerProtocolCalls.java:
##########
@@ -73,6 +75,50 @@ public final class ContainerProtocolCalls  {
   private ContainerProtocolCalls() {
   }
 
+  /**
+   * Calls the container protocol to list blocks in container.
+   *
+   * @param xceiverClient client to perform call
+   * @param containerID the ID of the container to list block
+   * @param replicaIndex the index of the replica in pipeline
+   * @param startLocalID the localID of the first block to get
+   * @param count max number of blocks to get
+   * @param token a token for this block (may be null)
+   * @return container protocol list block response
+   * @throws IOException if there is an I/O error while performing the call
+   */
+  public static ListBlockResponseProto listBlock(XceiverClientSpi xceiverClient,
+      long containerID, int replicaIndex, Long startLocalID, int count,
+      Token<? extends TokenIdentifier> token) throws IOException {
+
+    ListBlockRequestProto.Builder listBlockBuilder =
+        ListBlockRequestProto.newBuilder()
+            .setCount(count);
+
+    if (startLocalID != null) {
+      listBlockBuilder.setStartLocalID(startLocalID);
+    }
+
+    String datanodeID = xceiverClient.getPipeline().getNodesInOrder()
+        .get(replicaIndex).getUuidString();
+
+    ContainerCommandRequestProto.Builder builder =
+        ContainerCommandRequestProto.newBuilder()
+            .setCmdType(Type.ListBlock)
+            .setContainerID(containerID)
+            .setDatanodeUuid(datanodeID)
+            .setListBlock(listBlockBuilder.build());
+
+    if (token != null) {
+      builder.setEncodedToken(token.encodeToUrlString());
+    }
+
+    ContainerCommandRequestProto request = builder.build();
+    ContainerCommandResponseProto response =
+        xceiverClient.sendCommand(request, getValidatorList());

Review Comment:
   I just realized and it's good that server side ListBlock already handled.
   
   I think we can create singleECBlockPipeline with the current target node, so that the strategy will just work as it will have only one node. 
   
   Example, we did that in ECBlockOutputStreamEntry#createSingleECBlockPipeline



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r874274366


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {

Review Comment:
   Should this test be placed at `org.apache.hadoop.ozone.container.ec.reconstruction` then?
   Do you think there is a better package for it?



##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {
+
+  private static MiniOzoneCluster cluster;
+  private static OzoneManager om;
+  private static StorageContainerManager scm;
+  private static OzoneClient client;
+  private static ObjectStore store;
+  private static StorageContainerLocationProtocolClientSideTranslatorPB
+      storageContainerLocationClient;
+  private static final String SCM_ID = UUID.randomUUID().toString();
+  private static final String CLUSTER_ID = UUID.randomUUID().toString();
+  private static final int EC_DATA = 10;
+  private static final int EC_PARITY = 4;
+  private static final EcCodec EC_CODEC = EcCodec.RS;
+  private static final int EC_CHUNK_SIZE = 1024;
+  private static final int NUM_DN = EC_DATA + EC_PARITY;
+  private static final int[] KEY_SIZES = new int[]{
+      EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * 3 / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE * 3 / 2,
+  };
+  private static byte[][] values;
+  private static long containerID;
+  private static Pipeline pipeline;
+  private static List<DatanodeDetails> datanodeDetails;
+  private List<XceiverClientSpi> clients = null;
+
+  @BeforeAll
+  public static void init() throws Exception {
+    OzoneConfiguration conf = new OzoneConfiguration();
+    conf.setInt(ScmConfigKeys.OZONE_SCM_PIPELINE_OWNER_CONTAINER_COUNT, 1);
+    conf.setBoolean(OzoneConfigKeys.OZONE_ACL_ENABLED, true);
+    conf.set(OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS,
+        OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS_NATIVE);
+    startCluster(conf);
+    prepareData(KEY_SIZES);
+  }
+
+  @AfterAll
+  public static void stop() throws IOException {
+    stopCluster();
+  }
+
+  private Pipeline createSingleNodePipeline(Pipeline ecPipeline,
+      DatanodeDetails node, int replicaIndex) {
+
+    Map<DatanodeDetails, Integer> indicesForSinglePipeline = new HashMap<>();
+    indicesForSinglePipeline.put(node, replicaIndex);
+
+    return Pipeline.newBuilder()
+        .setId(ecPipeline.getId())
+        .setReplicationConfig(ecPipeline.getReplicationConfig())
+        .setState(ecPipeline.getPipelineState())
+        .setNodes(ImmutableList.of(node))
+        .setReplicaIndexes(indicesForSinglePipeline)
+        .build();
+  }
+
+  @BeforeEach
+  public void connectToDatanodes() {
+    clients = new ArrayList<>(datanodeDetails.size());
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      clients.add(new XceiverClientGrpc(
+          createSingleNodePipeline(pipeline, datanodeDetails.get(i), i + 1),
+          cluster.getConf()));
+    }
+  }
+
+  @AfterEach
+  public void closeClients() {
+    if (clients == null) {
+      return;
+    }
+    for (XceiverClientSpi c : clients) {
+      c.close();
+    }
+    clients = null;
+  }
+
+  private Function<Integer, Integer> chunksInReplicaFunc(int i) {
+    if (i < EC_DATA) {
+      return (keySize) -> {
+        int dataBlocks = (keySize + EC_CHUNK_SIZE - 1) / EC_CHUNK_SIZE;
+        return (dataBlocks + EC_DATA - 1 - i) / EC_DATA;
+      };
+    } else {
+      return (keySize) -> (keySize + EC_CHUNK_SIZE * EC_DATA - 1) /
+          (EC_CHUNK_SIZE * EC_DATA);
+    }
+  }
+
+  @Test
+  public void testListBlock() throws Exception {
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      final int minKeySize = i < EC_DATA ? i * EC_CHUNK_SIZE : 0;
+      int numExpectedBlocks = (int) Arrays.stream(KEY_SIZES)
+          .filter(size -> size > minKeySize).count();
+      Function<Integer, Integer> expectedChunksFunc = chunksInReplicaFunc(i);
+      int numExpectedChunks = Arrays.stream(KEY_SIZES)
+          .map(expectedChunksFunc::apply).sum();
+      try {
+        ListBlockResponseProto response = ContainerProtocolCalls.listBlock(
+            clients.get(i), containerID, null, numExpectedBlocks + 1, null);
+        Assertions.assertEquals(numExpectedBlocks, response.getBlockDataCount(),
+            "blocks count doesn't match on DN " + i);
+        Assertions.assertEquals(numExpectedChunks,
+            response.getBlockDataList().stream()
+                .mapToInt(BlockData::getChunksCount).sum(),
+            "chunks count doesn't match on DN " + i);
+      } catch (StorageContainerException e) {
+        Assertions.assertTrue(e.getMessage().contains("does not exist"));
+        Assertions.assertEquals(0, numExpectedBlocks);
+      }
+    }
+  }
+
+  public static void startCluster(OzoneConfiguration conf) throws Exception {
+
+    // Set minimum pipeline to 1 to ensure all data is written to
+    // the same container group
+    WritableECContainerProviderConfig writableECContainerProviderConfig =
+        conf.getObject(WritableECContainerProviderConfig.class);
+    writableECContainerProviderConfig.setMinimumPipelines(1);
+    conf.setFromObject(writableECContainerProviderConfig);
+
+    cluster = MiniOzoneCluster.newBuilder(conf)
+        .setNumDatanodes(NUM_DN)
+        .setScmId(SCM_ID)
+        .setClusterId(CLUSTER_ID)
+        .build();
+    cluster.waitForClusterToBeReady();
+    om = cluster.getOzoneManager();

Review Comment:
   Sure.



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r872949574


##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/ContainerProtocolCalls.java:
##########
@@ -73,6 +75,50 @@ public final class ContainerProtocolCalls  {
   private ContainerProtocolCalls() {
   }
 
+  /**
+   * Calls the container protocol to list blocks in container.
+   *
+   * @param xceiverClient client to perform call
+   * @param containerID the ID of the container to list block
+   * @param replicaIndex the index of the replica in pipeline
+   * @param startLocalID the localID of the first block to get
+   * @param count max number of blocks to get
+   * @param token a token for this block (may be null)
+   * @return container protocol list block response
+   * @throws IOException if there is an I/O error while performing the call
+   */
+  public static ListBlockResponseProto listBlock(XceiverClientSpi xceiverClient,
+      long containerID, int replicaIndex, Long startLocalID, int count,
+      Token<? extends TokenIdentifier> token) throws IOException {
+
+    ListBlockRequestProto.Builder listBlockBuilder =
+        ListBlockRequestProto.newBuilder()
+            .setCount(count);
+
+    if (startLocalID != null) {
+      listBlockBuilder.setStartLocalID(startLocalID);
+    }
+
+    String datanodeID = xceiverClient.getPipeline().getNodesInOrder()
+        .get(replicaIndex).getUuidString();
+
+    ContainerCommandRequestProto.Builder builder =
+        ContainerCommandRequestProto.newBuilder()
+            .setCmdType(Type.ListBlock)
+            .setContainerID(containerID)
+            .setDatanodeUuid(datanodeID)
+            .setListBlock(listBlockBuilder.build());
+
+    if (token != null) {
+      builder.setEncodedToken(token.encodeToUrlString());
+    }
+
+    ContainerCommandRequestProto request = builder.build();
+    ContainerCommandResponseProto response =
+        xceiverClient.sendCommand(request, getValidatorList());

Review Comment:
   > I think we can create singleECBlockPipeline with the current target node, so that the strategy will just work as it will have only one node.
   > 
   > Example, we did that in ECBlockOutputStreamEntry#createSingleECBlockPipeline
   
   Good idea, thanks for the advice.



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r874275928


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {
+
+  private static MiniOzoneCluster cluster;
+  private static OzoneManager om;
+  private static StorageContainerManager scm;
+  private static OzoneClient client;
+  private static ObjectStore store;
+  private static StorageContainerLocationProtocolClientSideTranslatorPB
+      storageContainerLocationClient;
+  private static final String SCM_ID = UUID.randomUUID().toString();
+  private static final String CLUSTER_ID = UUID.randomUUID().toString();
+  private static final int EC_DATA = 10;
+  private static final int EC_PARITY = 4;
+  private static final EcCodec EC_CODEC = EcCodec.RS;
+  private static final int EC_CHUNK_SIZE = 1024;
+  private static final int NUM_DN = EC_DATA + EC_PARITY;
+  private static final int[] KEY_SIZES = new int[]{
+      EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * 3 / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE * 3 / 2,
+  };
+  private static byte[][] values;
+  private static long containerID;
+  private static Pipeline pipeline;
+  private static List<DatanodeDetails> datanodeDetails;
+  private List<XceiverClientSpi> clients = null;
+
+  @BeforeAll
+  public static void init() throws Exception {
+    OzoneConfiguration conf = new OzoneConfiguration();
+    conf.setInt(ScmConfigKeys.OZONE_SCM_PIPELINE_OWNER_CONTAINER_COUNT, 1);
+    conf.setBoolean(OzoneConfigKeys.OZONE_ACL_ENABLED, true);
+    conf.set(OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS,
+        OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS_NATIVE);
+    startCluster(conf);
+    prepareData(KEY_SIZES);
+  }
+
+  @AfterAll
+  public static void stop() throws IOException {
+    stopCluster();
+  }
+
+  private Pipeline createSingleNodePipeline(Pipeline ecPipeline,
+      DatanodeDetails node, int replicaIndex) {
+
+    Map<DatanodeDetails, Integer> indicesForSinglePipeline = new HashMap<>();
+    indicesForSinglePipeline.put(node, replicaIndex);
+
+    return Pipeline.newBuilder()
+        .setId(ecPipeline.getId())
+        .setReplicationConfig(ecPipeline.getReplicationConfig())
+        .setState(ecPipeline.getPipelineState())
+        .setNodes(ImmutableList.of(node))
+        .setReplicaIndexes(indicesForSinglePipeline)
+        .build();
+  }
+
+  @BeforeEach
+  public void connectToDatanodes() {
+    clients = new ArrayList<>(datanodeDetails.size());
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      clients.add(new XceiverClientGrpc(
+          createSingleNodePipeline(pipeline, datanodeDetails.get(i), i + 1),
+          cluster.getConf()));
+    }
+  }
+
+  @AfterEach
+  public void closeClients() {
+    if (clients == null) {
+      return;
+    }
+    for (XceiverClientSpi c : clients) {
+      c.close();
+    }
+    clients = null;
+  }
+
+  private Function<Integer, Integer> chunksInReplicaFunc(int i) {
+    if (i < EC_DATA) {
+      return (keySize) -> {
+        int dataBlocks = (keySize + EC_CHUNK_SIZE - 1) / EC_CHUNK_SIZE;
+        return (dataBlocks + EC_DATA - 1 - i) / EC_DATA;
+      };
+    } else {
+      return (keySize) -> (keySize + EC_CHUNK_SIZE * EC_DATA - 1) /
+          (EC_CHUNK_SIZE * EC_DATA);
+    }
+  }
+
+  @Test
+  public void testListBlock() throws Exception {
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      final int minKeySize = i < EC_DATA ? i * EC_CHUNK_SIZE : 0;
+      int numExpectedBlocks = (int) Arrays.stream(KEY_SIZES)
+          .filter(size -> size > minKeySize).count();
+      Function<Integer, Integer> expectedChunksFunc = chunksInReplicaFunc(i);
+      int numExpectedChunks = Arrays.stream(KEY_SIZES)
+          .map(expectedChunksFunc::apply).sum();
+      try {
+        ListBlockResponseProto response = ContainerProtocolCalls.listBlock(
+            clients.get(i), containerID, null, numExpectedBlocks + 1, null);
+        Assertions.assertEquals(numExpectedBlocks, response.getBlockDataCount(),
+            "blocks count doesn't match on DN " + i);
+        Assertions.assertEquals(numExpectedChunks,
+            response.getBlockDataList().stream()
+                .mapToInt(BlockData::getChunksCount).sum(),
+            "chunks count doesn't match on DN " + i);
+      } catch (StorageContainerException e) {

Review Comment:
   For example, if we change `KEY_SIZES` to `new int[]{EC_CHUNK_SIZE / 2}`.



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] umamaheswararao commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
umamaheswararao commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r875220213


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {

Review Comment:
   As all commands implemented at org.apache.hadoop.hdds.scm.storage
   It make sense to have a package with that in hdds common?



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] umamaheswararao commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
umamaheswararao commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r874248895


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {

Review Comment:
   Probably we should rename this to TestContainerListBlockCommand?  or TestContainerCommands ?
   I don't think we are actually testing EC Reconstruction in this tests. 



##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {
+
+  private static MiniOzoneCluster cluster;
+  private static OzoneManager om;
+  private static StorageContainerManager scm;
+  private static OzoneClient client;
+  private static ObjectStore store;
+  private static StorageContainerLocationProtocolClientSideTranslatorPB
+      storageContainerLocationClient;
+  private static final String SCM_ID = UUID.randomUUID().toString();
+  private static final String CLUSTER_ID = UUID.randomUUID().toString();
+  private static final int EC_DATA = 10;
+  private static final int EC_PARITY = 4;
+  private static final EcCodec EC_CODEC = EcCodec.RS;
+  private static final int EC_CHUNK_SIZE = 1024;
+  private static final int NUM_DN = EC_DATA + EC_PARITY;
+  private static final int[] KEY_SIZES = new int[]{
+      EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * 3 / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE * 3 / 2,
+  };
+  private static byte[][] values;
+  private static long containerID;
+  private static Pipeline pipeline;
+  private static List<DatanodeDetails> datanodeDetails;
+  private List<XceiverClientSpi> clients = null;
+
+  @BeforeAll
+  public static void init() throws Exception {
+    OzoneConfiguration conf = new OzoneConfiguration();
+    conf.setInt(ScmConfigKeys.OZONE_SCM_PIPELINE_OWNER_CONTAINER_COUNT, 1);
+    conf.setBoolean(OzoneConfigKeys.OZONE_ACL_ENABLED, true);
+    conf.set(OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS,
+        OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS_NATIVE);
+    startCluster(conf);
+    prepareData(KEY_SIZES);
+  }
+
+  @AfterAll
+  public static void stop() throws IOException {
+    stopCluster();
+  }
+
+  private Pipeline createSingleNodePipeline(Pipeline ecPipeline,
+      DatanodeDetails node, int replicaIndex) {
+
+    Map<DatanodeDetails, Integer> indicesForSinglePipeline = new HashMap<>();
+    indicesForSinglePipeline.put(node, replicaIndex);
+
+    return Pipeline.newBuilder()
+        .setId(ecPipeline.getId())
+        .setReplicationConfig(ecPipeline.getReplicationConfig())
+        .setState(ecPipeline.getPipelineState())
+        .setNodes(ImmutableList.of(node))
+        .setReplicaIndexes(indicesForSinglePipeline)
+        .build();
+  }
+
+  @BeforeEach
+  public void connectToDatanodes() {
+    clients = new ArrayList<>(datanodeDetails.size());
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      clients.add(new XceiverClientGrpc(
+          createSingleNodePipeline(pipeline, datanodeDetails.get(i), i + 1),
+          cluster.getConf()));
+    }
+  }
+
+  @AfterEach
+  public void closeClients() {
+    if (clients == null) {
+      return;
+    }
+    for (XceiverClientSpi c : clients) {
+      c.close();
+    }
+    clients = null;
+  }
+
+  private Function<Integer, Integer> chunksInReplicaFunc(int i) {
+    if (i < EC_DATA) {
+      return (keySize) -> {
+        int dataBlocks = (keySize + EC_CHUNK_SIZE - 1) / EC_CHUNK_SIZE;
+        return (dataBlocks + EC_DATA - 1 - i) / EC_DATA;
+      };
+    } else {
+      return (keySize) -> (keySize + EC_CHUNK_SIZE * EC_DATA - 1) /
+          (EC_CHUNK_SIZE * EC_DATA);
+    }
+  }
+
+  @Test
+  public void testListBlock() throws Exception {
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      final int minKeySize = i < EC_DATA ? i * EC_CHUNK_SIZE : 0;
+      int numExpectedBlocks = (int) Arrays.stream(KEY_SIZES)
+          .filter(size -> size > minKeySize).count();
+      Function<Integer, Integer> expectedChunksFunc = chunksInReplicaFunc(i);
+      int numExpectedChunks = Arrays.stream(KEY_SIZES)
+          .map(expectedChunksFunc::apply).sum();
+      try {
+        ListBlockResponseProto response = ContainerProtocolCalls.listBlock(
+            clients.get(i), containerID, null, numExpectedBlocks + 1, null);
+        Assertions.assertEquals(numExpectedBlocks, response.getBlockDataCount(),
+            "blocks count doesn't match on DN " + i);
+        Assertions.assertEquals(numExpectedChunks,
+            response.getBlockDataList().stream()
+                .mapToInt(BlockData::getChunksCount).sum(),
+            "chunks count doesn't match on DN " + i);
+      } catch (StorageContainerException e) {

Review Comment:
   Isn't it this StorageContainerException is a failure? Why do we need to assert. We should not get the exception in ideal case right?



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] umamaheswararao commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
umamaheswararao commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r874252618


##########
hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ec/reconstruction/TestECReconstructionCommands.java:
##########
@@ -0,0 +1,249 @@
+/*
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.hadoop.ozone.container.ec.reconstruction;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.hdds.client.ECReplicationConfig;
+import org.apache.hadoop.hdds.client.ECReplicationConfig.EcCodec;
+import org.apache.hadoop.hdds.client.ReplicationConfig;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.BlockData;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ListBlockResponseProto;
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.XceiverClientSpi;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.common.helpers.StorageContainerException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.WritableECContainerProvider.WritableECContainerProviderConfig;
+import org.apache.hadoop.hdds.scm.protocolPB.StorageContainerLocationProtocolClientSideTranslatorPB;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.hdds.scm.storage.ContainerProtocolCalls;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.client.ObjectStore;
+import org.apache.hadoop.ozone.client.OzoneBucket;
+import org.apache.hadoop.ozone.client.OzoneClient;
+import org.apache.hadoop.ozone.client.OzoneClientFactory;
+import org.apache.hadoop.ozone.client.OzoneVolume;
+import org.apache.hadoop.ozone.om.OzoneManager;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import java.io.IOException;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.UUID;
+import java.util.function.Function;
+
+/**
+ * This class tests commands used in EC reconstruction.
+ */
+public class TestECReconstructionCommands {
+
+  private static MiniOzoneCluster cluster;
+  private static OzoneManager om;
+  private static StorageContainerManager scm;
+  private static OzoneClient client;
+  private static ObjectStore store;
+  private static StorageContainerLocationProtocolClientSideTranslatorPB
+      storageContainerLocationClient;
+  private static final String SCM_ID = UUID.randomUUID().toString();
+  private static final String CLUSTER_ID = UUID.randomUUID().toString();
+  private static final int EC_DATA = 10;
+  private static final int EC_PARITY = 4;
+  private static final EcCodec EC_CODEC = EcCodec.RS;
+  private static final int EC_CHUNK_SIZE = 1024;
+  private static final int NUM_DN = EC_DATA + EC_PARITY;
+  private static final int[] KEY_SIZES = new int[]{
+      EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * 3 / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE / 2,
+      EC_CHUNK_SIZE * EC_DATA + EC_CHUNK_SIZE * 3 / 2,
+  };
+  private static byte[][] values;
+  private static long containerID;
+  private static Pipeline pipeline;
+  private static List<DatanodeDetails> datanodeDetails;
+  private List<XceiverClientSpi> clients = null;
+
+  @BeforeAll
+  public static void init() throws Exception {
+    OzoneConfiguration conf = new OzoneConfiguration();
+    conf.setInt(ScmConfigKeys.OZONE_SCM_PIPELINE_OWNER_CONTAINER_COUNT, 1);
+    conf.setBoolean(OzoneConfigKeys.OZONE_ACL_ENABLED, true);
+    conf.set(OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS,
+        OzoneConfigKeys.OZONE_ACL_AUTHORIZER_CLASS_NATIVE);
+    startCluster(conf);
+    prepareData(KEY_SIZES);
+  }
+
+  @AfterAll
+  public static void stop() throws IOException {
+    stopCluster();
+  }
+
+  private Pipeline createSingleNodePipeline(Pipeline ecPipeline,
+      DatanodeDetails node, int replicaIndex) {
+
+    Map<DatanodeDetails, Integer> indicesForSinglePipeline = new HashMap<>();
+    indicesForSinglePipeline.put(node, replicaIndex);
+
+    return Pipeline.newBuilder()
+        .setId(ecPipeline.getId())
+        .setReplicationConfig(ecPipeline.getReplicationConfig())
+        .setState(ecPipeline.getPipelineState())
+        .setNodes(ImmutableList.of(node))
+        .setReplicaIndexes(indicesForSinglePipeline)
+        .build();
+  }
+
+  @BeforeEach
+  public void connectToDatanodes() {
+    clients = new ArrayList<>(datanodeDetails.size());
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      clients.add(new XceiverClientGrpc(
+          createSingleNodePipeline(pipeline, datanodeDetails.get(i), i + 1),
+          cluster.getConf()));
+    }
+  }
+
+  @AfterEach
+  public void closeClients() {
+    if (clients == null) {
+      return;
+    }
+    for (XceiverClientSpi c : clients) {
+      c.close();
+    }
+    clients = null;
+  }
+
+  private Function<Integer, Integer> chunksInReplicaFunc(int i) {
+    if (i < EC_DATA) {
+      return (keySize) -> {
+        int dataBlocks = (keySize + EC_CHUNK_SIZE - 1) / EC_CHUNK_SIZE;
+        return (dataBlocks + EC_DATA - 1 - i) / EC_DATA;
+      };
+    } else {
+      return (keySize) -> (keySize + EC_CHUNK_SIZE * EC_DATA - 1) /
+          (EC_CHUNK_SIZE * EC_DATA);
+    }
+  }
+
+  @Test
+  public void testListBlock() throws Exception {
+    for (int i = 0; i < datanodeDetails.size(); i++) {
+      final int minKeySize = i < EC_DATA ? i * EC_CHUNK_SIZE : 0;
+      int numExpectedBlocks = (int) Arrays.stream(KEY_SIZES)
+          .filter(size -> size > minKeySize).count();
+      Function<Integer, Integer> expectedChunksFunc = chunksInReplicaFunc(i);
+      int numExpectedChunks = Arrays.stream(KEY_SIZES)
+          .map(expectedChunksFunc::apply).sum();
+      try {
+        ListBlockResponseProto response = ContainerProtocolCalls.listBlock(
+            clients.get(i), containerID, null, numExpectedBlocks + 1, null);
+        Assertions.assertEquals(numExpectedBlocks, response.getBlockDataCount(),
+            "blocks count doesn't match on DN " + i);
+        Assertions.assertEquals(numExpectedChunks,
+            response.getBlockDataList().stream()
+                .mapToInt(BlockData::getChunksCount).sum(),
+            "chunks count doesn't match on DN " + i);
+      } catch (StorageContainerException e) {
+        Assertions.assertTrue(e.getMessage().contains("does not exist"));
+        Assertions.assertEquals(0, numExpectedBlocks);
+      }
+    }
+  }
+
+  public static void startCluster(OzoneConfiguration conf) throws Exception {
+
+    // Set minimum pipeline to 1 to ensure all data is written to
+    // the same container group
+    WritableECContainerProviderConfig writableECContainerProviderConfig =
+        conf.getObject(WritableECContainerProviderConfig.class);
+    writableECContainerProviderConfig.setMinimumPipelines(1);
+    conf.setFromObject(writableECContainerProviderConfig);
+
+    cluster = MiniOzoneCluster.newBuilder(conf)
+        .setNumDatanodes(NUM_DN)
+        .setScmId(SCM_ID)
+        .setClusterId(CLUSTER_ID)
+        .build();
+    cluster.waitForClusterToBeReady();
+    om = cluster.getOzoneManager();

Review Comment:
   I think we are not using all of this variable right? ex: om. If we are not using let's get ride of them, we can add when 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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on a diff in pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on code in PR #3410:
URL: https://github.com/apache/ozone/pull/3410#discussion_r872165629


##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/ContainerProtocolCalls.java:
##########
@@ -73,6 +75,50 @@ public final class ContainerProtocolCalls  {
   private ContainerProtocolCalls() {
   }
 
+  /**
+   * Calls the container protocol to list blocks in container.
+   *
+   * @param xceiverClient client to perform call
+   * @param containerID the ID of the container to list block
+   * @param replicaIndex the index of the replica in pipeline
+   * @param startLocalID the localID of the first block to get
+   * @param count max number of blocks to get
+   * @param token a token for this block (may be null)
+   * @return container protocol list block response
+   * @throws IOException if there is an I/O error while performing the call
+   */
+  public static ListBlockResponseProto listBlock(XceiverClientSpi xceiverClient,
+      long containerID, int replicaIndex, Long startLocalID, int count,
+      Token<? extends TokenIdentifier> token) throws IOException {
+
+    ListBlockRequestProto.Builder listBlockBuilder =
+        ListBlockRequestProto.newBuilder()
+            .setCount(count);
+
+    if (startLocalID != null) {
+      listBlockBuilder.setStartLocalID(startLocalID);
+    }
+
+    String datanodeID = xceiverClient.getPipeline().getNodesInOrder()
+        .get(replicaIndex).getUuidString();
+
+    ContainerCommandRequestProto.Builder builder =
+        ContainerCommandRequestProto.newBuilder()
+            .setCmdType(Type.ListBlock)
+            .setContainerID(containerID)
+            .setDatanodeUuid(datanodeID)
+            .setListBlock(listBlockBuilder.build());
+
+    if (token != null) {
+      builder.setEncodedToken(token.encodeToUrlString());
+    }
+
+    ContainerCommandRequestProto request = builder.build();
+    ContainerCommandResponseProto response =
+        xceiverClient.sendCommand(request, getValidatorList());

Review Comment:
   We may need to call `xceiverClient#sendCommandOnAllNodes` here because of the `sendCommand` node picking strategy. What do you think @umamaheswararao?
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L457-L458
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L141-L142
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/client/src/main/java/org/apache/hadoop/hdds/scm/XceiverClientGrpc.java#L154-L155



##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/ContainerProtocolCalls.java:
##########
@@ -73,6 +75,50 @@ public final class ContainerProtocolCalls  {
   private ContainerProtocolCalls() {
   }
 
+  /**
+   * Calls the container protocol to list blocks in container.
+   *
+   * @param xceiverClient client to perform call
+   * @param containerID the ID of the container to list block
+   * @param replicaIndex the index of the replica in pipeline
+   * @param startLocalID the localID of the first block to get
+   * @param count max number of blocks to get
+   * @param token a token for this block (may be null)
+   * @return container protocol list block response
+   * @throws IOException if there is an I/O error while performing the call
+   */
+  public static ListBlockResponseProto listBlock(XceiverClientSpi xceiverClient,
+      long containerID, int replicaIndex, Long startLocalID, int count,
+      Token<? extends TokenIdentifier> token) throws IOException {
+
+    ListBlockRequestProto.Builder listBlockBuilder =
+        ListBlockRequestProto.newBuilder()
+            .setCount(count);
+
+    if (startLocalID != null) {
+      listBlockBuilder.setStartLocalID(startLocalID);
+    }
+
+    String datanodeID = xceiverClient.getPipeline().getNodesInOrder()
+        .get(replicaIndex).getUuidString();
+
+    ContainerCommandRequestProto.Builder builder =
+        ContainerCommandRequestProto.newBuilder()
+            .setCmdType(Type.ListBlock)
+            .setContainerID(containerID)
+            .setDatanodeUuid(datanodeID)

Review Comment:
   Seems setting `DatanodeUUID` here is useless, this is the only place where it is read.
   
   https://github.com/apache/ozone/blob/96669856ad44a7571b1dd4267f28f37055b9e4b1/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/impl/HddsDispatcher.java#L423



-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] umamaheswararao merged pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
umamaheswararao merged PR #3410:
URL: https://github.com/apache/ozone/pull/3410


-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org


[GitHub] [ozone] kaijchen commented on pull request #3410: HDDS-6596. EC: Support ListBlock from CoordinatorDN

Posted by GitBox <gi...@apache.org>.
kaijchen commented on PR #3410:
URL: https://github.com/apache/ozone/pull/3410#issuecomment-1130977052

   Thanks @umamaheswararao for the review.


-- 
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: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org
For additional commands, e-mail: issues-help@ozone.apache.org