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/10/11 08:32:52 UTC

[GitHub] [ozone] kaijchen commented on a diff in pull request #3811: HDDS-7300. Race condition between full data scan and block deletion

kaijchen commented on code in PR #3811:
URL: https://github.com/apache/ozone/pull/3811#discussion_r992007985


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/keyvalue/KeyValueContainerCheck.java:
##########
@@ -230,44 +232,100 @@ private void scanData(DataTransferThrottler throttler, Canceler canceler)
 
     onDiskContainerData.setDbFile(dbFile);
 
-    ContainerLayoutVersion layout = onDiskContainerData.getLayoutVersion();
-
     try (DBHandle db = BlockUtils.getDB(onDiskContainerData, checkConfig);
         BlockIterator<BlockData> kvIter = db.getStore().getBlockIterator(
             onDiskContainerData.getContainerID(),
             onDiskContainerData.getUnprefixedKeyFilter())) {
 
       while (kvIter.hasNext()) {
         BlockData block = kvIter.nextBlock();
-        for (ContainerProtos.ChunkInfo chunk : block.getChunks()) {
-          File chunkFile = layout.getChunkFile(onDiskContainerData,
-              block.getBlockID(), ChunkInfo.getFromProtoBuf(chunk));
-
-          if (!chunkFile.exists()) {
-            // concurrent mutation in Block DB? lookup the block again.
-            String blockKey =
-                onDiskContainerData.blockKey(block.getBlockID().getLocalID());
-            BlockData bdata = db.getStore()
-                    .getBlockDataTable()
-                    .get(blockKey);
-            // In EC, client may write empty putBlock in padding block nodes.
-            // So, we need to make sure, chunk length > 0, before declaring
-            // the missing chunk file.
-            if (bdata != null && bdata.getChunks().size() > 0 && bdata
-                .getChunks().get(0).getLen() > 0) {
-              throw new IOException(
-                  "Missing chunk file " + chunkFile.getAbsolutePath());
+
+        // If holding read lock for the entire duration, including wait() calls
+        // in DataTransferThrottler, would effectively make other threads
+        // throttled.
+        // Here try optimistically and retry with the container lock to
+        // make sure reading the latest record. If the record is just removed,
+        // the block should be skipped to scan.
+        try {
+          scanBlock(db, block, throttler, canceler);
+        } catch (IOException ex) {
+          if (getBlockDataFromDBWithLock(db, block) == null) {
+            if (LOG.isDebugEnabled()) {
+              LOG.debug("Scanned outdated blockData {} in container {}.",
+                  block, containerID);
             }
-          } else if (chunk.getChecksumData().getType()
-              != ContainerProtos.ChecksumType.NONE) {
-            verifyChecksum(block, chunk, chunkFile, layout, throttler,
-                canceler);
+          } else {
+            throw new IOException(ex);
           }
         }
       }
     }
   }
 
+  /**
+   *  Attempt to read the block data without the container lock.
+   *  The block onDisk might be in modification by other thread and not yet
+   *  flushed to DB, so the content might be outdated.
+   *
+   * @param db DB of container
+   * @param block last queried blockData
+   * @return blockData in DB
+   * @throws IOException
+   */
+  private BlockData getBlockDataFromDB(DBHandle db, BlockData block)
+      throws IOException {
+    String blockKey =
+        onDiskContainerData.blockKey(block.getBlockID().getLocalID());
+    return db.getStore().getBlockDataTable().get(blockKey);
+  }
+
+  /**
+   *  Attempt to read the block data with the container lock.
+   *  The container lock ensure the latest DB record could be retrieved, since
+   *  other block related write operation will acquire the container write lock.
+   *
+   * @param db DB of container
+   * @param block last queried blockData
+   * @return blockData in DB
+   * @throws IOException
+   */
+  private BlockData getBlockDataFromDBWithLock(DBHandle db, BlockData block)
+      throws IOException {
+    container.readLock();
+    try {
+      return getBlockDataFromDB(db, block);
+    } finally {
+      container.readUnlock();
+    }
+  }
+
+  private void scanBlock(DBHandle db, BlockData block,
+      DataTransferThrottler throttler, Canceler canceler) throws IOException {
+    ContainerLayoutVersion layout = onDiskContainerData.getLayoutVersion();
+
+    for (ContainerProtos.ChunkInfo chunk : block.getChunks()) {
+      File chunkFile = layout.getChunkFile(onDiskContainerData,
+          block.getBlockID(), ChunkInfo.getFromProtoBuf(chunk));
+
+      if (!chunkFile.exists()) {
+        // concurrent mutation in Block DB? lookup the block again.
+        BlockData bdata = getBlockDataFromDB(db, block);
+        // In EC, client may write empty putBlock in padding block nodes.
+        // So, we need to make sure, chunk length > 0, before declaring
+        // the missing chunk file.
+        if (bdata != null && bdata.getChunks().size() > 0 && bdata
+            .getChunks().get(0).getLen() > 0) {
+          throw new IOException(

Review Comment:
   Can we check `getBlockDataFromDBWithLock() == null` here before throwing exception?
   
   PS: Do we need a lock to check DB for the second time here?



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