You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@ozone.apache.org by "vtutrinov (via GitHub)" <gi...@apache.org> on 2023/12/04 10:39:13 UTC

[PR] HDDS-9807. Take into account the hdds volume committed bytes on detecting the datanode availability to allocate a new container on SCM side [ozone]

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

   ## What changes were proposed in this pull request?
   
   The algorithm of detection of the datanode availability for a new container was different between the SCM and datanodes in case of EC buckets. The SCM doesn't take into account the committed bytes count and serves the client with an invalid pipeline and then the datanodes from the provided pipeline will fail to create new containers
   
   The PR proposes a common way of computing the datanode availability for a new container allocation
   
   ## What is the link to the Apache JIRA
   https://issues.apache.org/jira/browse/HDDS-9807
   
   ## How was this patch tested?
   * manually in a way described in the jira ticket,
   * new unit test that checks the datanode is not available to allocate a new container in case of increasing committed bytes count
   


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "symious (via GitHub)" <gi...@apache.org>.
symious commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1866378512

   @adoroszlai Thank you for the confirmation.


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1421694975


##########
hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestSCMCommonPlacementPolicy.java:
##########
@@ -448,11 +461,68 @@ protected List<DatanodeDetails> chooseDatanodesInternal(
           }
         };
     dummyPlacementPolicy.chooseDatanodes(null, null, 1, 1, 1);
-    Assertions.assertFalse(usedNodesIdentity.get());
+    assertFalse(usedNodesIdentity.get());
     dummyPlacementPolicy.chooseDatanodes(null, null, null, 1, 1, 1);
     Assertions.assertTrue(usedNodesIdentity.get());
   }
 
+  @Test
+  public void testDatanodeIsInvalidIsCaseOfIncreasingCommittedBytes() {
+    NodeManager nodeMngr = mock(NodeManager.class);
+    ConfigurationSource confing = mock(ConfigurationSource.class);
+    when(confing.isConfigured(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE)))
+        .thenReturn(true);
+    when(confing.getStorageSize(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE),
+            eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE_DEFAULT),
+            eq(StorageUnit.BYTES))).thenReturn(100000.0);
+    UUID datanodeUuid = spy(UUID.randomUUID());
+    DummyPlacementPolicy placementPolicy =
+        new DummyPlacementPolicy(nodeMngr, confing, 1);
+    DatanodeDetails datanodeDetails = mock(DatanodeDetails.class);
+    when(datanodeDetails.getUuid()).thenReturn(datanodeUuid);
+
+    DatanodeInfo datanodeInfo = mock(DatanodeInfo.class);
+    NodeStatus nodeStatus = mock(NodeStatus.class);
+    when(nodeStatus.isNodeWritable()).thenReturn(true);
+    when(datanodeInfo.getNodeStatus()).thenReturn(nodeStatus);
+    when(nodeMngr.getNodeByUuid(eq(datanodeUuid))).thenReturn(datanodeInfo);
+
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport1 =
+        StorageContainerDatanodeProtocolProtos.StorageReportProto.newBuilder()
+            .setCommitted(500)
+            .setCapacity(200000)
+            .setRemaining(101000)
+            .setStorageUuid(UUID.randomUUID().toString())
+            .setStorageLocation("/data/hdds")
+            .build();
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport2 =
+        StorageContainerDatanodeProtocolProtos.StorageReportProto.newBuilder()
+            .setCommitted(1000)
+            .setCapacity(200000)
+            .setRemaining(101000)
+            .setStorageUuid(UUID.randomUUID().toString())
+            .setStorageLocation("/data/hdds")
+            .build();
+    StorageContainerDatanodeProtocolProtos.MetadataStorageReportProto
+        metaReport =
+        StorageContainerDatanodeProtocolProtos.MetadataStorageReportProto
+            .newBuilder()
+            .setRemaining(200)
+            .setStorageLocation("/data/metadata")
+            .build();
+    when(datanodeInfo.getStorageReports())
+        .thenReturn(Collections.singletonList(storageReport1))
+        .thenReturn(Collections.singletonList(storageReport2));
+    when(datanodeInfo.getMetadataStorageReports())
+        .thenReturn(Collections.singletonList(metaReport));
+
+
+    // 500 committed bytes
+    assertTrue(placementPolicy.isValidNode(datanodeDetails, 100, 4000));

Review Comment:
   I don't agree that it would be correct. We check the placement policy's behavior here and its implementations (e.g. org.apache.hadoop.hdds.scm.container.placement.algorithms.SCMContainerPlacementRackScatter) use **isValid** method to check the datanode availability



##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/volume/AvailableSpaceFilter.java:
##########
@@ -70,6 +70,17 @@ public String toString() {
         ", volumes: " + fullVolumes;
   }
 
+  public static boolean hasVolumeEnoughSpace(long volumeCapacity,

Review Comment:
   Done



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1421694502


##########
hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestSCMCommonPlacementPolicy.java:
##########
@@ -448,11 +461,68 @@ protected List<DatanodeDetails> chooseDatanodesInternal(
           }
         };
     dummyPlacementPolicy.chooseDatanodes(null, null, 1, 1, 1);
-    Assertions.assertFalse(usedNodesIdentity.get());
+    assertFalse(usedNodesIdentity.get());
     dummyPlacementPolicy.chooseDatanodes(null, null, null, 1, 1, 1);
     Assertions.assertTrue(usedNodesIdentity.get());
   }
 
+  @Test
+  public void testDatanodeIsInvalidIsCaseOfIncreasingCommittedBytes() {
+    NodeManager nodeMngr = mock(NodeManager.class);
+    ConfigurationSource confing = mock(ConfigurationSource.class);
+    when(confing.isConfigured(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE)))
+        .thenReturn(true);
+    when(confing.getStorageSize(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE),
+            eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE_DEFAULT),
+            eq(StorageUnit.BYTES))).thenReturn(100000.0);
+    UUID datanodeUuid = spy(UUID.randomUUID());

Review Comment:
   no, it isn't, fixed



##########
hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestSCMCommonPlacementPolicy.java:
##########
@@ -448,11 +461,68 @@ protected List<DatanodeDetails> chooseDatanodesInternal(
           }
         };
     dummyPlacementPolicy.chooseDatanodes(null, null, 1, 1, 1);
-    Assertions.assertFalse(usedNodesIdentity.get());
+    assertFalse(usedNodesIdentity.get());
     dummyPlacementPolicy.chooseDatanodes(null, null, null, 1, 1, 1);
     Assertions.assertTrue(usedNodesIdentity.get());
   }
 
+  @Test
+  public void testDatanodeIsInvalidIsCaseOfIncreasingCommittedBytes() {
+    NodeManager nodeMngr = mock(NodeManager.class);
+    ConfigurationSource confing = mock(ConfigurationSource.class);
+    when(confing.isConfigured(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE)))
+        .thenReturn(true);
+    when(confing.getStorageSize(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE),
+            eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE_DEFAULT),
+            eq(StorageUnit.BYTES))).thenReturn(100000.0);
+    UUID datanodeUuid = spy(UUID.randomUUID());
+    DummyPlacementPolicy placementPolicy =
+        new DummyPlacementPolicy(nodeMngr, confing, 1);
+    DatanodeDetails datanodeDetails = mock(DatanodeDetails.class);
+    when(datanodeDetails.getUuid()).thenReturn(datanodeUuid);
+
+    DatanodeInfo datanodeInfo = mock(DatanodeInfo.class);
+    NodeStatus nodeStatus = mock(NodeStatus.class);
+    when(nodeStatus.isNodeWritable()).thenReturn(true);
+    when(datanodeInfo.getNodeStatus()).thenReturn(nodeStatus);
+    when(nodeMngr.getNodeByUuid(eq(datanodeUuid))).thenReturn(datanodeInfo);
+
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport1 =

Review Comment:
   Done



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1422388981


##########
hadoop-hdds/tools/src/main/java/org/apache/hadoop/hdds/scm/cli/datanode/UsageInfoSubcommand.java:
##########
@@ -155,6 +155,14 @@ private void printInfo(DatanodeUsage info) {
         + " B", StringUtils.byteDesc(info.getRemaining()));
     System.out.printf("%-13s: %s %n", "Remaining %",
         PERCENT_FORMAT.format(info.getRemainingRatio()));
+    System.out.printf("%-13s: %s (%s) %n", "Container Pre-allocated",
+        info.getCommitted() + " B", StringUtils.byteDesc(info.getCommitted()));
+    System.out.printf("%-13s: %s (%s) %n", "Remaining Allocatable",
+        (info.getRemaining() - info.getCommitted()) + " B",
+        StringUtils.byteDesc((info.getRemaining() - info.getCommitted())));
+    System.out.printf("%-13s: %s (%s) %n", "Free Space To Spare",

Review Comment:
   Done



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1417602407


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/interfaces/StorageLocationReportMXBean.java:
##########
@@ -33,6 +33,8 @@ public interface StorageLocationReportMXBean {
 
   long getRemaining();

Review Comment:
   Exactly, and that's why I would provide the committed bytes count in reports (e.g. in datanode heartbeat requests, **datanode usageifo** or in **datanode list** cli commands) as a separate field and don't mix it with the remaining bytes count (**Devide et impera** policy :) )



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "xichen01 (via GitHub)" <gi...@apache.org>.
xichen01 commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1415504662


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/interfaces/StorageLocationReportMXBean.java:
##########
@@ -33,6 +33,8 @@ public interface StorageLocationReportMXBean {
 
   long getRemaining();

Review Comment:
   @vtutrinov @adoroszlai 
   What is the definition of `remaining`, I noticed that `remaining` is actually Volume `available`. `available=capacity - reserved - used`.
   
   - So does `committed` have to be reported to the SCM separately.
   - Or is it added to `remaining`, which is defined as the amount of space on the Volume that is guaranteed to be available for data storage? `available=capacity - reserved - used - commited`.
   We use `available` in a lot of places, and it seems like we should take `commited` into `available`.
   
   
   https://github.com/apache/ozone/blob/9b3d045a9a2d87b67cc8523d166c3a82c4e05fa5/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/volume/MutableVolumeSet.java#L473



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1415715892


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/interfaces/StorageLocationReportMXBean.java:
##########
@@ -33,6 +33,8 @@ public interface StorageLocationReportMXBean {
 
   long getRemaining();

Review Comment:
   seems it makes sense, I will try to check the approach



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "sumitagrawl (via GitHub)" <gi...@apache.org>.
sumitagrawl commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1422119388


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/volume/AvailableSpaceFilter.java:
##########
@@ -42,10 +42,10 @@ public boolean test(HddsVolume vol) {
     long free = vol.getAvailable();
     long committed = vol.getCommittedBytes();
     long available = free - committed;
-    long volumeFreeSpace =
+    long volumeFreeSpaceTpSpare =

Review Comment:
   nit: spelling incorrect, volumeFreeSpaceToSpare



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "adoroszlai (via GitHub)" <gi...@apache.org>.
adoroszlai commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1850196818

   @aswinshakil @sadanand48 please 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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "myskov (via GitHub)" <gi...@apache.org>.
myskov commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1862393085

   @aswinshakil @sadanand48 please 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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "symious (via GitHub)" <gi...@apache.org>.
symious commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1866304440

   @adoroszlai Could you help to confirm if this PR needs to be cherry-picked to ozone-1.4?


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "xichen01 (via GitHub)" <gi...@apache.org>.
xichen01 commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1420709857


##########
hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestSCMCommonPlacementPolicy.java:
##########
@@ -448,11 +461,68 @@ protected List<DatanodeDetails> chooseDatanodesInternal(
           }
         };
     dummyPlacementPolicy.chooseDatanodes(null, null, 1, 1, 1);
-    Assertions.assertFalse(usedNodesIdentity.get());
+    assertFalse(usedNodesIdentity.get());
     dummyPlacementPolicy.chooseDatanodes(null, null, null, 1, 1, 1);
     Assertions.assertTrue(usedNodesIdentity.get());
   }
 
+  @Test
+  public void testDatanodeIsInvalidIsCaseOfIncreasingCommittedBytes() {
+    NodeManager nodeMngr = mock(NodeManager.class);
+    ConfigurationSource confing = mock(ConfigurationSource.class);
+    when(confing.isConfigured(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE)))
+        .thenReturn(true);
+    when(confing.getStorageSize(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE),
+            eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE_DEFAULT),
+            eq(StorageUnit.BYTES))).thenReturn(100000.0);
+    UUID datanodeUuid = spy(UUID.randomUUID());
+    DummyPlacementPolicy placementPolicy =
+        new DummyPlacementPolicy(nodeMngr, confing, 1);
+    DatanodeDetails datanodeDetails = mock(DatanodeDetails.class);
+    when(datanodeDetails.getUuid()).thenReturn(datanodeUuid);
+
+    DatanodeInfo datanodeInfo = mock(DatanodeInfo.class);
+    NodeStatus nodeStatus = mock(NodeStatus.class);
+    when(nodeStatus.isNodeWritable()).thenReturn(true);
+    when(datanodeInfo.getNodeStatus()).thenReturn(nodeStatus);
+    when(nodeMngr.getNodeByUuid(eq(datanodeUuid))).thenReturn(datanodeInfo);
+
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport1 =

Review Comment:
   nit: can consider using the `org.apache.hadoop.hdds.scm.HddsTestUtils#createStorageReport`



##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/volume/AvailableSpaceFilter.java:
##########
@@ -70,6 +70,17 @@ public String toString() {
         ", volumes: " + fullVolumes;
   }
 
+  public static boolean hasVolumeEnoughSpace(long volumeCapacity,

Review Comment:
   Can put it into `VolumeUsage`, just like `getMinVolumeFreeSpace`



##########
hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestSCMCommonPlacementPolicy.java:
##########
@@ -448,11 +461,68 @@ protected List<DatanodeDetails> chooseDatanodesInternal(
           }
         };
     dummyPlacementPolicy.chooseDatanodes(null, null, 1, 1, 1);
-    Assertions.assertFalse(usedNodesIdentity.get());
+    assertFalse(usedNodesIdentity.get());
     dummyPlacementPolicy.chooseDatanodes(null, null, null, 1, 1, 1);
     Assertions.assertTrue(usedNodesIdentity.get());
   }
 
+  @Test
+  public void testDatanodeIsInvalidIsCaseOfIncreasingCommittedBytes() {
+    NodeManager nodeMngr = mock(NodeManager.class);
+    ConfigurationSource confing = mock(ConfigurationSource.class);
+    when(confing.isConfigured(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE)))
+        .thenReturn(true);
+    when(confing.getStorageSize(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE),
+            eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE_DEFAULT),
+            eq(StorageUnit.BYTES))).thenReturn(100000.0);
+    UUID datanodeUuid = spy(UUID.randomUUID());
+    DummyPlacementPolicy placementPolicy =
+        new DummyPlacementPolicy(nodeMngr, confing, 1);
+    DatanodeDetails datanodeDetails = mock(DatanodeDetails.class);
+    when(datanodeDetails.getUuid()).thenReturn(datanodeUuid);
+
+    DatanodeInfo datanodeInfo = mock(DatanodeInfo.class);
+    NodeStatus nodeStatus = mock(NodeStatus.class);
+    when(nodeStatus.isNodeWritable()).thenReturn(true);
+    when(datanodeInfo.getNodeStatus()).thenReturn(nodeStatus);
+    when(nodeMngr.getNodeByUuid(eq(datanodeUuid))).thenReturn(datanodeInfo);
+
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport1 =
+        StorageContainerDatanodeProtocolProtos.StorageReportProto.newBuilder()
+            .setCommitted(500)
+            .setCapacity(200000)
+            .setRemaining(101000)
+            .setStorageUuid(UUID.randomUUID().toString())
+            .setStorageLocation("/data/hdds")
+            .build();
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport2 =
+        StorageContainerDatanodeProtocolProtos.StorageReportProto.newBuilder()
+            .setCommitted(1000)
+            .setCapacity(200000)
+            .setRemaining(101000)
+            .setStorageUuid(UUID.randomUUID().toString())
+            .setStorageLocation("/data/hdds")
+            .build();
+    StorageContainerDatanodeProtocolProtos.MetadataStorageReportProto
+        metaReport =
+        StorageContainerDatanodeProtocolProtos.MetadataStorageReportProto
+            .newBuilder()
+            .setRemaining(200)
+            .setStorageLocation("/data/metadata")
+            .build();
+    when(datanodeInfo.getStorageReports())
+        .thenReturn(Collections.singletonList(storageReport1))
+        .thenReturn(Collections.singletonList(storageReport2));
+    when(datanodeInfo.getMetadataStorageReports())
+        .thenReturn(Collections.singletonList(metaReport));
+
+
+    // 500 committed bytes

Review Comment:
   nit: It would be good to comment on the process of calculating.



##########
hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestSCMCommonPlacementPolicy.java:
##########
@@ -448,11 +461,68 @@ protected List<DatanodeDetails> chooseDatanodesInternal(
           }
         };
     dummyPlacementPolicy.chooseDatanodes(null, null, 1, 1, 1);
-    Assertions.assertFalse(usedNodesIdentity.get());
+    assertFalse(usedNodesIdentity.get());
     dummyPlacementPolicy.chooseDatanodes(null, null, null, 1, 1, 1);
     Assertions.assertTrue(usedNodesIdentity.get());
   }
 
+  @Test
+  public void testDatanodeIsInvalidIsCaseOfIncreasingCommittedBytes() {
+    NodeManager nodeMngr = mock(NodeManager.class);
+    ConfigurationSource confing = mock(ConfigurationSource.class);
+    when(confing.isConfigured(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE)))
+        .thenReturn(true);
+    when(confing.getStorageSize(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE),
+            eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE_DEFAULT),
+            eq(StorageUnit.BYTES))).thenReturn(100000.0);
+    UUID datanodeUuid = spy(UUID.randomUUID());

Review Comment:
   `spy` Is it necessary for UUID?



##########
hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestSCMCommonPlacementPolicy.java:
##########
@@ -448,11 +461,68 @@ protected List<DatanodeDetails> chooseDatanodesInternal(
           }
         };
     dummyPlacementPolicy.chooseDatanodes(null, null, 1, 1, 1);
-    Assertions.assertFalse(usedNodesIdentity.get());
+    assertFalse(usedNodesIdentity.get());
     dummyPlacementPolicy.chooseDatanodes(null, null, null, 1, 1, 1);
     Assertions.assertTrue(usedNodesIdentity.get());
   }
 
+  @Test
+  public void testDatanodeIsInvalidIsCaseOfIncreasingCommittedBytes() {
+    NodeManager nodeMngr = mock(NodeManager.class);
+    ConfigurationSource confing = mock(ConfigurationSource.class);
+    when(confing.isConfigured(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE)))
+        .thenReturn(true);
+    when(confing.getStorageSize(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE),
+            eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE_DEFAULT),
+            eq(StorageUnit.BYTES))).thenReturn(100000.0);
+    UUID datanodeUuid = spy(UUID.randomUUID());
+    DummyPlacementPolicy placementPolicy =
+        new DummyPlacementPolicy(nodeMngr, confing, 1);
+    DatanodeDetails datanodeDetails = mock(DatanodeDetails.class);
+    when(datanodeDetails.getUuid()).thenReturn(datanodeUuid);
+
+    DatanodeInfo datanodeInfo = mock(DatanodeInfo.class);
+    NodeStatus nodeStatus = mock(NodeStatus.class);
+    when(nodeStatus.isNodeWritable()).thenReturn(true);
+    when(datanodeInfo.getNodeStatus()).thenReturn(nodeStatus);
+    when(nodeMngr.getNodeByUuid(eq(datanodeUuid))).thenReturn(datanodeInfo);
+
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport1 =
+        StorageContainerDatanodeProtocolProtos.StorageReportProto.newBuilder()
+            .setCommitted(500)
+            .setCapacity(200000)
+            .setRemaining(101000)
+            .setStorageUuid(UUID.randomUUID().toString())
+            .setStorageLocation("/data/hdds")
+            .build();
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport2 =
+        StorageContainerDatanodeProtocolProtos.StorageReportProto.newBuilder()
+            .setCommitted(1000)
+            .setCapacity(200000)
+            .setRemaining(101000)
+            .setStorageUuid(UUID.randomUUID().toString())
+            .setStorageLocation("/data/hdds")
+            .build();
+    StorageContainerDatanodeProtocolProtos.MetadataStorageReportProto
+        metaReport =
+        StorageContainerDatanodeProtocolProtos.MetadataStorageReportProto
+            .newBuilder()
+            .setRemaining(200)
+            .setStorageLocation("/data/metadata")
+            .build();
+    when(datanodeInfo.getStorageReports())
+        .thenReturn(Collections.singletonList(storageReport1))
+        .thenReturn(Collections.singletonList(storageReport2));
+    when(datanodeInfo.getMetadataStorageReports())
+        .thenReturn(Collections.singletonList(metaReport));
+
+
+    // 500 committed bytes
+    assertTrue(placementPolicy.isValidNode(datanodeDetails, 100, 4000));

Review Comment:
   nit: can directly use `hasEnoughSpace` to assert



##########
hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/SCMCommonPlacementPolicy.java:
##########
@@ -308,7 +312,9 @@ public static boolean hasEnoughSpace(DatanodeDetails datanodeDetails,
 
     if (dataSizeRequired > 0) {
       for (StorageReportProto reportProto : datanodeInfo.getStorageReports()) {
-        if (reportProto.getRemaining() > dataSizeRequired) {
+        if (hasVolumeEnoughSpace(reportProto.getCapacity(),
+            reportProto.getRemaining(), reportProto.getCommitted(),
+            dataSizeRequired, conf)) {

Review Comment:
   The configuration used in `getMinVolumeFreeSpace` is the Datanode configuration, what if the configuration is different for different Datanodes or not available in SCM?
   Maybe `volumeFreeSpaceToSpare` should also report



##########
hadoop-hdds/tools/src/main/java/org/apache/hadoop/hdds/scm/cli/datanode/UsageInfoSubcommand.java:
##########
@@ -155,6 +155,8 @@ private void printInfo(DatanodeUsage info) {
         + " B", StringUtils.byteDesc(info.getRemaining()));
     System.out.printf("%-13s: %s %n", "Remaining %",
         PERCENT_FORMAT.format(info.getRemainingRatio()));
+    System.out.printf("%-13s: %s (%s) %n", "Committed", info.getCommitted()

Review Comment:
   `Committed` doesn't seem to be easy to understand outside of the code context, suggesting: `Container Preallocated`. 
   And add a `Remaining Allocatable` (remaining - commited)



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1422197420


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/volume/AvailableSpaceFilter.java:
##########
@@ -42,10 +42,10 @@ public boolean test(HddsVolume vol) {
     long free = vol.getAvailable();
     long committed = vol.getCommittedBytes();
     long available = free - committed;
-    long volumeFreeSpace =
+    long volumeFreeSpaceTpSpare =

Review Comment:
   Fixed



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1417203747


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/interfaces/StorageLocationReportMXBean.java:
##########
@@ -33,6 +33,8 @@ public interface StorageLocationReportMXBean {
 
   long getRemaining();

Review Comment:
   @xichen01 
   I've pushed a new commit with the proposed approach and the build is green, as such as the manual test from the jira ticket. But I'm concerned about the datanode storage metrics (output of the **ozone admin datanode usageinfo** command) and recon's reports - from the user's perspective it will seen contr ordinal:
   
   * I've written a 200KiB key to the EC-bucket with rs-6-3-1024k replication (the cluster has 10 datanodes with 2Gb storages mounted to /data)
   * datanodes report before the key was written:
   
   {code}
   Usage Information (1 Datanodes)
   
   UUID         : 8ac06e73-a434-4fd3-9be5-bc271b8f6bf7 
   IP Address   : 172.23.0.15 
   Hostname     : ozone-datanode10-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 9e3bf76a-8c98-4fa3-b034-c68973d7560b 
   IP Address   : 172.23.0.10 
   Hostname     : ozone-datanode3-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : a5df4b7d-92b3-407a-b56d-aaeb924f0cd6 
   IP Address   : 172.23.0.7 
   Hostname     : ozone-datanode2-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 27365376 B (26.10 MB) 
   Total Used % : 1.27% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2120118272 B (1.97 GB) 
   Remaining %  : 98.73% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : e5a630fc-ce77-4211-89dc-15ab0ca67fde 
   IP Address   : 172.23.0.6 
   Hostname     : ozone-datanode8-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 2d00f163-7f22-4395-b82b-e8353c74698b 
   IP Address   : 172.23.0.5 
   Hostname     : ozone-datanode7-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 3a22dc09-acc5-4270-8007-135056031ed9 
   IP Address   : 172.23.0.12 
   Hostname     : ozone-datanode9-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 4478a423-0416-429e-b41c-5e58b8ec7ed5 
   IP Address   : 172.23.0.16 
   Hostname     : ozone-datanode5-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 5ffda4af-e5dd-4b11-a945-7fef0b53356b 
   IP Address   : 172.23.0.17 
   Hostname     : ozone-datanode1-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 6fe5b830-5031-4a4c-a8fa-ae909c55d187 
   IP Address   : 172.23.0.3 
   Hostname     : ozone-datanode6-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 7ce23e7c-41bf-4c5e-aea7-63af06049ad0 
   IP Address   : 172.23.0.19 
   Hostname     : ozone-datanode4-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   {code}
   
   * datanode report after the key was written:
   
   {code}
   Usage Information (1 Datanodes)
   
   UUID         : 8ac06e73-a434-4fd3-9be5-bc271b8f6bf7 
   IP Address   : 172.23.0.15 
   Hostname     : ozone-datanode10-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 9e3bf76a-8c98-4fa3-b034-c68973d7560b 
   IP Address   : 172.23.0.10 
   Hostname     : ozone-datanode3-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : a5df4b7d-92b3-407a-b56d-aaeb924f0cd6 
   IP Address   : 172.23.0.7 
   Hostname     : ozone-datanode2-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1248776192 B (1.16 GB) 
   Total Used % : 58.15% 
   Ozone Used   : 204800 B (200 KB) 
   Ozone Used % : 0.01% 
   Remaining    : 898707456 B (857.07 MB) 
   Remaining %  : 41.85% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : e5a630fc-ce77-4211-89dc-15ab0ca67fde 
   IP Address   : 172.23.0.6 
   Hostname     : ozone-datanode8-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31596544 B (30.13 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115887104 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 2d00f163-7f22-4395-b82b-e8353c74698b 
   IP Address   : 172.23.0.5 
   Hostname     : ozone-datanode7-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 204800 B (200 KB) 
   Ozone Used % : 0.01% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 3a22dc09-acc5-4270-8007-135056031ed9 
   IP Address   : 172.23.0.12 
   Hostname     : ozone-datanode9-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 4478a423-0416-429e-b41c-5e58b8ec7ed5 
   IP Address   : 172.23.0.16 
   Hostname     : ozone-datanode5-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 5ffda4af-e5dd-4b11-a945-7fef0b53356b 
   IP Address   : 172.23.0.17 
   Hostname     : ozone-datanode1-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 204800 B (200 KB) 
   Ozone Used % : 0.01% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 6fe5b830-5031-4a4c-a8fa-ae909c55d187 
   IP Address   : 172.23.0.3 
   Hostname     : ozone-datanode6-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 204800 B (200 KB) 
   Ozone Used % : 0.01% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 7ce23e7c-41bf-4c5e-aea7-63af06049ad0 
   IP Address   : 172.23.0.19 
   Hostname     : ozone-datanode4-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   {code}
   
   WHAT ??? 200KiB written key has changed the remaining storage size of 9 datanodes from 1.97GiB to 853MB, looks too strange, doesn't it?
   I would report the committed bytes as a separate report data field as the remaining storage size will be more obvious for the user and the cluster admins



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "xichen01 (via GitHub)" <gi...@apache.org>.
xichen01 commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1422267003


##########
hadoop-hdds/tools/src/main/java/org/apache/hadoop/hdds/scm/cli/datanode/UsageInfoSubcommand.java:
##########
@@ -155,6 +155,14 @@ private void printInfo(DatanodeUsage info) {
         + " B", StringUtils.byteDesc(info.getRemaining()));
     System.out.printf("%-13s: %s %n", "Remaining %",
         PERCENT_FORMAT.format(info.getRemainingRatio()));
+    System.out.printf("%-13s: %s (%s) %n", "Container Pre-allocated",
+        info.getCommitted() + " B", StringUtils.byteDesc(info.getCommitted()));
+    System.out.printf("%-13s: %s (%s) %n", "Remaining Allocatable",
+        (info.getRemaining() - info.getCommitted()) + " B",
+        StringUtils.byteDesc((info.getRemaining() - info.getCommitted())));
+    System.out.printf("%-13s: %s (%s) %n", "Free Space To Spare",

Review Comment:
   We should make this newly added lines aligned
   Such as:
   ```bash
   //..
   Remaining    : 4751560704 B (4.43 GB)
   Remaining %  : 45.02%
   Container Pre-allocated : 0 B (0 B)
   Remaining Allocatable   : 4751560704 B (4.43 GB)
   Free Space To Spare     : 0 B (0 B)
   ```



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1848897401

   > Thanks @vtutrinov for updating the patch. There is a test failure in `TestSCMCommonPlacementPolicy.testDatanodeIsInvalidInCaseOfIncreasingCommittedBytes`: https://github.com/vtutrinov/ozone/actions/runs/7156214054/job/19485712139#step:5:1750
   
   Fixed


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "xichen01 (via GitHub)" <gi...@apache.org>.
xichen01 commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1419422181


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/interfaces/StorageLocationReportMXBean.java:
##########
@@ -33,6 +33,8 @@ public interface StorageLocationReportMXBean {
 
   long getRemaining();

Review Comment:
   Yes, it looks like your initial solution is the more appropriate choice.



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "adoroszlai (via GitHub)" <gi...@apache.org>.
adoroszlai commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1864857546

   Thanks @vtutrinov for the patch, @sumitagrawl, @xichen01 for the review.
   
   (Since release branch for 1.4 has been cut, we can fix any post-commit comments later, without holding back the upcoming release.)


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1417203747


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/interfaces/StorageLocationReportMXBean.java:
##########
@@ -33,6 +33,8 @@ public interface StorageLocationReportMXBean {
 
   long getRemaining();

Review Comment:
   @xichen01 
   I've pushed a new commit with the proposed approach and the build is green, as such as the manual test from the jira ticket. But I'm concerned about the datanode storage metrics (output of the **ozone admin datanode usageinfo** command) and recon's reports - from the user's perspective it will seen contr ordinal:
   
   * I've written a 200KiB key to the EC-bucket with rs-6-3-1024k replication (the cluster has 10 datanodes with 2Gb storages mounted to /data)
   * datanodes report before the key was written:
   
   ```
   Usage Information (1 Datanodes)
   
   UUID         : 8ac06e73-a434-4fd3-9be5-bc271b8f6bf7 
   IP Address   : 172.23.0.15 
   Hostname     : ozone-datanode10-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 9e3bf76a-8c98-4fa3-b034-c68973d7560b 
   IP Address   : 172.23.0.10 
   Hostname     : ozone-datanode3-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : a5df4b7d-92b3-407a-b56d-aaeb924f0cd6 
   IP Address   : 172.23.0.7 
   Hostname     : ozone-datanode2-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 27365376 B (26.10 MB) 
   Total Used % : 1.27% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2120118272 B (1.97 GB) 
   Remaining %  : 98.73% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : e5a630fc-ce77-4211-89dc-15ab0ca67fde 
   IP Address   : 172.23.0.6 
   Hostname     : ozone-datanode8-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 2d00f163-7f22-4395-b82b-e8353c74698b 
   IP Address   : 172.23.0.5 
   Hostname     : ozone-datanode7-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 3a22dc09-acc5-4270-8007-135056031ed9 
   IP Address   : 172.23.0.12 
   Hostname     : ozone-datanode9-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 4478a423-0416-429e-b41c-5e58b8ec7ed5 
   IP Address   : 172.23.0.16 
   Hostname     : ozone-datanode5-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 5ffda4af-e5dd-4b11-a945-7fef0b53356b 
   IP Address   : 172.23.0.17 
   Hostname     : ozone-datanode1-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 6fe5b830-5031-4a4c-a8fa-ae909c55d187 
   IP Address   : 172.23.0.3 
   Hostname     : ozone-datanode6-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 7ce23e7c-41bf-4c5e-aea7-63af06049ad0 
   IP Address   : 172.23.0.19 
   Hostname     : ozone-datanode4-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31571968 B (30.11 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115911680 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   ```
   
   * datanode report after the key was written:
   
   ```
   Usage Information (1 Datanodes)
   
   UUID         : 8ac06e73-a434-4fd3-9be5-bc271b8f6bf7 
   IP Address   : 172.23.0.15 
   Hostname     : ozone-datanode10-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 9e3bf76a-8c98-4fa3-b034-c68973d7560b 
   IP Address   : 172.23.0.10 
   Hostname     : ozone-datanode3-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : a5df4b7d-92b3-407a-b56d-aaeb924f0cd6 
   IP Address   : 172.23.0.7 
   Hostname     : ozone-datanode2-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1248776192 B (1.16 GB) 
   Total Used % : 58.15% 
   Ozone Used   : 204800 B (200 KB) 
   Ozone Used % : 0.01% 
   Remaining    : 898707456 B (857.07 MB) 
   Remaining %  : 41.85% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : e5a630fc-ce77-4211-89dc-15ab0ca67fde 
   IP Address   : 172.23.0.6 
   Hostname     : ozone-datanode8-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 31596544 B (30.13 MB) 
   Total Used % : 1.47% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 2115887104 B (1.97 GB) 
   Remaining %  : 98.53% 
   Container(s) : 0 
   
   Usage Information (1 Datanodes)
   
   UUID         : 2d00f163-7f22-4395-b82b-e8353c74698b 
   IP Address   : 172.23.0.5 
   Hostname     : ozone-datanode7-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 204800 B (200 KB) 
   Ozone Used % : 0.01% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 3a22dc09-acc5-4270-8007-135056031ed9 
   IP Address   : 172.23.0.12 
   Hostname     : ozone-datanode9-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 4478a423-0416-429e-b41c-5e58b8ec7ed5 
   IP Address   : 172.23.0.16 
   Hostname     : ozone-datanode5-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 5ffda4af-e5dd-4b11-a945-7fef0b53356b 
   IP Address   : 172.23.0.17 
   Hostname     : ozone-datanode1-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 204800 B (200 KB) 
   Ozone Used % : 0.01% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 6fe5b830-5031-4a4c-a8fa-ae909c55d187 
   IP Address   : 172.23.0.3 
   Hostname     : ozone-datanode6-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 204800 B (200 KB) 
   Ozone Used % : 0.01% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   
   Usage Information (1 Datanodes)
   
   UUID         : 7ce23e7c-41bf-4c5e-aea7-63af06049ad0 
   IP Address   : 172.23.0.19 
   Hostname     : ozone-datanode4-1.ozone_default 
   Capacity     : 2147483648 B (2 GB) 
   Total Used   : 1252982784 B (1.17 GB) 
   Total Used % : 58.35% 
   Ozone Used   : 0 B (0 B) 
   Ozone Used % : 0.00% 
   Remaining    : 894500864 B (853.06 MB) 
   Remaining %  : 41.65% 
   Container(s) : 1 
   ```
   
   WHAT ??? 200KiB written key has changed the remaining storage size of 9 datanodes from 1.97GiB to 853MB, looks too strange, doesn't it?
   I would report the committed bytes as a separate report data field as the remaining storage size will be more obvious for the user and the cluster admins



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1421694521


##########
hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestSCMCommonPlacementPolicy.java:
##########
@@ -448,11 +461,68 @@ protected List<DatanodeDetails> chooseDatanodesInternal(
           }
         };
     dummyPlacementPolicy.chooseDatanodes(null, null, 1, 1, 1);
-    Assertions.assertFalse(usedNodesIdentity.get());
+    assertFalse(usedNodesIdentity.get());
     dummyPlacementPolicy.chooseDatanodes(null, null, null, 1, 1, 1);
     Assertions.assertTrue(usedNodesIdentity.get());
   }
 
+  @Test
+  public void testDatanodeIsInvalidIsCaseOfIncreasingCommittedBytes() {
+    NodeManager nodeMngr = mock(NodeManager.class);
+    ConfigurationSource confing = mock(ConfigurationSource.class);
+    when(confing.isConfigured(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE)))
+        .thenReturn(true);
+    when(confing.getStorageSize(eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE),
+            eq(HDDS_DATANODE_VOLUME_MIN_FREE_SPACE_DEFAULT),
+            eq(StorageUnit.BYTES))).thenReturn(100000.0);
+    UUID datanodeUuid = spy(UUID.randomUUID());
+    DummyPlacementPolicy placementPolicy =
+        new DummyPlacementPolicy(nodeMngr, confing, 1);
+    DatanodeDetails datanodeDetails = mock(DatanodeDetails.class);
+    when(datanodeDetails.getUuid()).thenReturn(datanodeUuid);
+
+    DatanodeInfo datanodeInfo = mock(DatanodeInfo.class);
+    NodeStatus nodeStatus = mock(NodeStatus.class);
+    when(nodeStatus.isNodeWritable()).thenReturn(true);
+    when(datanodeInfo.getNodeStatus()).thenReturn(nodeStatus);
+    when(nodeMngr.getNodeByUuid(eq(datanodeUuid))).thenReturn(datanodeInfo);
+
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport1 =
+        StorageContainerDatanodeProtocolProtos.StorageReportProto.newBuilder()
+            .setCommitted(500)
+            .setCapacity(200000)
+            .setRemaining(101000)
+            .setStorageUuid(UUID.randomUUID().toString())
+            .setStorageLocation("/data/hdds")
+            .build();
+    StorageContainerDatanodeProtocolProtos.StorageReportProto storageReport2 =
+        StorageContainerDatanodeProtocolProtos.StorageReportProto.newBuilder()
+            .setCommitted(1000)
+            .setCapacity(200000)
+            .setRemaining(101000)
+            .setStorageUuid(UUID.randomUUID().toString())
+            .setStorageLocation("/data/hdds")
+            .build();
+    StorageContainerDatanodeProtocolProtos.MetadataStorageReportProto
+        metaReport =
+        StorageContainerDatanodeProtocolProtos.MetadataStorageReportProto
+            .newBuilder()
+            .setRemaining(200)
+            .setStorageLocation("/data/metadata")
+            .build();
+    when(datanodeInfo.getStorageReports())
+        .thenReturn(Collections.singletonList(storageReport1))
+        .thenReturn(Collections.singletonList(storageReport2));
+    when(datanodeInfo.getMetadataStorageReports())
+        .thenReturn(Collections.singletonList(metaReport));
+
+
+    // 500 committed bytes

Review Comment:
   Done



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1421694991


##########
hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/SCMCommonPlacementPolicy.java:
##########
@@ -308,7 +312,9 @@ public static boolean hasEnoughSpace(DatanodeDetails datanodeDetails,
 
     if (dataSizeRequired > 0) {
       for (StorageReportProto reportProto : datanodeInfo.getStorageReports()) {
-        if (reportProto.getRemaining() > dataSizeRequired) {
+        if (hasVolumeEnoughSpace(reportProto.getCapacity(),
+            reportProto.getRemaining(), reportProto.getCommitted(),
+            dataSizeRequired, conf)) {

Review Comment:
   Done



##########
hadoop-hdds/tools/src/main/java/org/apache/hadoop/hdds/scm/cli/datanode/UsageInfoSubcommand.java:
##########
@@ -155,6 +155,8 @@ private void printInfo(DatanodeUsage info) {
         + " B", StringUtils.byteDesc(info.getRemaining()));
     System.out.printf("%-13s: %s %n", "Remaining %",
         PERCENT_FORMAT.format(info.getRemainingRatio()));
+    System.out.printf("%-13s: %s (%s) %n", "Committed", info.getCommitted()

Review Comment:
   Done



-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "xichen01 (via GitHub)" <gi...@apache.org>.
xichen01 commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1850188729

   @vtutrinov Thank for updating the patch. LGTM + 1 .


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "xichen01 (via GitHub)" <gi...@apache.org>.
xichen01 commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1849017486

   @vtutrinov The patch looks good.
   The only thing we might want to consider at this point is `FreeSpaceToSpare`. Requires the "remaining allocatable space" greater than `FreeSpaceToSpare` when create a new `Container`, so maybe we should add that to the output of the `UsageInfo command` as well, what do you think?


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "adoroszlai (via GitHub)" <gi...@apache.org>.
adoroszlai commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1866375460

   @symious @myskov I suggest not including it in 1.4.0, as this had little time to be tested.  We can work on a patch realease 1.4.1 soon afterwards.


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "szetszwo (via GitHub)" <gi...@apache.org>.
szetszwo commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1871858439

   This may cause NPE; see HDDS-10027.


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "xichen01 (via GitHub)" <gi...@apache.org>.
xichen01 commented on code in PR #5721:
URL: https://github.com/apache/ozone/pull/5721#discussion_r1417576798


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/interfaces/StorageLocationReportMXBean.java:
##########
@@ -33,6 +33,8 @@ public interface StorageLocationReportMXBean {
 
   long getRemaining();

Review Comment:
   I think this is because we include `commited` in the `remaining`, so when a Container is created, the space that the Container might use will be deducted from the `remaining`.
   
   If we don't include `commited` in the `remaining`, the problem is that the `datanode list` command shows that there is enough space remaining to create a Container, but in reality it is not possible to create a new Container, because inside the Datanode, the creation of a Container has to take into account `commited`.



-- 
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


Re: [PR] HDDS-9807. Take into account the hdds volume committed bytes on detecting the datanode availability to allocate a new container on SCM side [ozone]

Posted by "adoroszlai (via GitHub)" <gi...@apache.org>.
adoroszlai commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1838552256

   @xichen01 please take a look


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "adoroszlai (via GitHub)" <gi...@apache.org>.
adoroszlai merged PR #5721:
URL: https://github.com/apache/ozone/pull/5721


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "adoroszlai (via GitHub)" <gi...@apache.org>.
adoroszlai commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1848889021

   Thanks @vtutrinov for updating the patch.  There is a test failure in `TestSCMCommonPlacementPolicy.testDatanodeIsInvalidInCaseOfIncreasingCommittedBytes`: https://github.com/vtutrinov/ozone/actions/runs/7156214054/job/19485712139#step:5:1750


-- 
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


Re: [PR] HDDS-9807. Consider volume committed space when checking if datanode can host new container [ozone]

Posted by "vtutrinov (via GitHub)" <gi...@apache.org>.
vtutrinov commented on PR #5721:
URL: https://github.com/apache/ozone/pull/5721#issuecomment-1849403783

   > @vtutrinov The patch looks good. The only thing we might want to consider at this point is `FreeSpaceToSpare`. Requires the "remaining allocatable space" greater than `FreeSpaceToSpare` when create a new `Container`, so maybe we should add that to the output of the `UsageInfo command` as well, what do you think?
   
   Yep, it makes sense, I will do


-- 
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