This is an automated email from the ASF dual-hosted git repository.

prashantpogde pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/ozone.git


The following commit(s) were added to refs/heads/master by this push:
     new 0862a76ef1 HDDS-8935. [Snapshot] Fallback to full diff if 
getDetlaFiles from compaction DAG fails (#4986)
0862a76ef1 is described below

commit 0862a76ef178f427a272c335374760c87c885119
Author: Hemant Kumar <[email protected]>
AuthorDate: Tue Jun 27 10:01:48 2023 -0700

    HDDS-8935. [Snapshot] Fallback to full diff if getDetlaFiles from 
compaction DAG fails (#4986)
---
 .../ozone/rocksdiff/RocksDBCheckpointDiffer.java   | 54 +++++++++---------
 .../org/apache/hadoop/ozone/om/TestOmSnapshot.java | 34 ++++++++---
 .../apache/hadoop/ozone/om/OmSnapshotManager.java  | 12 ++++
 .../ozone/om/snapshot/SnapshotDiffManager.java     | 13 +++--
 .../ozone/om/snapshot/TestSnapshotDiffManager.java | 66 ++++++++++++++++++++--
 5 files changed, 138 insertions(+), 41 deletions(-)

diff --git 
a/hadoop-hdds/rocksdb-checkpoint-differ/src/main/java/org/apache/ozone/rocksdiff/RocksDBCheckpointDiffer.java
 
b/hadoop-hdds/rocksdb-checkpoint-differ/src/main/java/org/apache/ozone/rocksdiff/RocksDBCheckpointDiffer.java
index 72f837fa85..4bac3978d1 100644
--- 
a/hadoop-hdds/rocksdb-checkpoint-differ/src/main/java/org/apache/ozone/rocksdiff/RocksDBCheckpointDiffer.java
+++ 
b/hadoop-hdds/rocksdb-checkpoint-differ/src/main/java/org/apache/ozone/rocksdiff/RocksDBCheckpointDiffer.java
@@ -858,27 +858,25 @@ public class RocksDBCheckpointDiffer implements 
AutoCloseable,
    *         e.g. ["/path/to/sstBackupDir/000050.sst",
    *               "/path/to/sstBackupDir/000060.sst"]
    */
-  public List<String> getSSTDiffListWithFullPath(
+  public synchronized List<String> getSSTDiffListWithFullPath(
       DifferSnapshotInfo src,
       DifferSnapshotInfo dest,
       String sstFilesDirForSnapDiffJob
   ) throws IOException {
 
-    synchronized (this) {
-      List<String> sstDiffList = getSSTDiffList(src, dest);
-
-      return sstDiffList.stream()
-          .map(
-              sst -> {
-                String sstFullPath = getSSTFullPath(sst, src.getDbPath());
-                Path link = Paths.get(sstFilesDirForSnapDiffJob,
-                    sst + SST_FILE_EXTENSION);
-                Path srcFile = Paths.get(sstFullPath);
-                createLink(link, srcFile);
-                return link.toString();
-              })
-          .collect(Collectors.toList());
-    }
+    List<String> sstDiffList = getSSTDiffList(src, dest);
+
+    return sstDiffList.stream()
+        .map(
+            sst -> {
+              String sstFullPath = getSSTFullPath(sst, src.getDbPath());
+              Path link = Paths.get(sstFilesDirForSnapDiffJob,
+                  sst + SST_FILE_EXTENSION);
+              Path srcFile = Paths.get(sstFullPath);
+              createLink(link, srcFile);
+              return link.toString();
+            })
+        .collect(Collectors.toList());
   }
 
   /**
@@ -892,8 +890,8 @@ public class RocksDBCheckpointDiffer implements 
AutoCloseable,
    * @param dest destination snapshot
    * @return A list of SST files without extension. e.g. ["000050", "000060"]
    */
-  public List<String> getSSTDiffList(DifferSnapshotInfo src,
-                                     DifferSnapshotInfo dest)
+  public synchronized List<String> getSSTDiffList(DifferSnapshotInfo src,
+                                                  DifferSnapshotInfo dest)
       throws IOException {
 
     // TODO: Reject or swap if dest is taken after src, once snapshot chain
@@ -963,7 +961,7 @@ public class RocksDBCheckpointDiffer implements 
AutoCloseable,
    * diffing).  Otherwise, add it to the differentFiles map, as it will
    * need further diffing.
    */
-  void internalGetSSTDiffList(
+  synchronized void internalGetSSTDiffList(
       DifferSnapshotInfo src, DifferSnapshotInfo dest,
       Set<String> srcSnapFiles, Set<String> destSnapFiles,
       MutableGraph<CompactionNode> mutableGraph,
@@ -1163,6 +1161,9 @@ public class RocksDBCheckpointDiffer implements 
AutoCloseable,
 
     Set<String> sstFileNodesRemoved =
         pruneSstFileNodesFromDag(lastCompactionSstFiles);
+
+    LOG.info("Removing SST files: {} as part of compaction DAG pruning.",
+        sstFileNodesRemoved);
     try (BootstrapStateHandler.Lock lock = getBootstrapStateLock().lock()) {
       removeSstFiles(sstFileNodesRemoved);
       deleteOlderSnapshotsCompactionFiles(olderSnapshotsLogFilePaths);
@@ -1463,14 +1464,15 @@ public class RocksDBCheckpointDiffer implements 
AutoCloseable,
    * those are not needed to generate snapshot diff. These files are basically
    * non-leaf nodes of the DAG.
    */
-  public void pruneSstFiles() {
+  public synchronized void pruneSstFiles() {
     Set<String> nonLeafSstFiles;
-    synchronized (this) {
-      nonLeafSstFiles = forwardCompactionDAG.nodes().stream()
-          .filter(node -> !forwardCompactionDAG.successors(node).isEmpty())
-          .map(node -> node.getFileName())
-          .collect(Collectors.toSet());
-    }
+    nonLeafSstFiles = forwardCompactionDAG.nodes().stream()
+        .filter(node -> !forwardCompactionDAG.successors(node).isEmpty())
+        .map(node -> node.getFileName())
+        .collect(Collectors.toSet());
+
+    LOG.info("Removing SST files: {} as part of SST file pruning.",
+        nonLeafSstFiles);
     try (BootstrapStateHandler.Lock lock = getBootstrapStateLock().lock()) {
       removeSstFiles(nonLeafSstFiles);
     } catch (InterruptedException e) {
diff --git 
a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/om/TestOmSnapshot.java
 
b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/om/TestOmSnapshot.java
index 2726ecc506..197376867a 100644
--- 
a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/om/TestOmSnapshot.java
+++ 
b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/om/TestOmSnapshot.java
@@ -91,6 +91,7 @@ import static 
org.apache.hadoop.ozone.om.OMConfigKeys.OZONE_OM_ENABLE_FILESYSTEM
 import static 
org.apache.hadoop.ozone.om.OMConfigKeys.OZONE_OM_SNAPSHOT_FORCE_FULL_DIFF;
 import static org.apache.hadoop.ozone.om.OmSnapshotManager.DELIMITER;
 import static 
org.apache.hadoop.ozone.om.exceptions.OMException.ResultCodes.CONTAINS_SNAPSHOT;
+import static 
org.apache.hadoop.ozone.om.exceptions.OMException.ResultCodes.INTERNAL_ERROR;
 import static 
org.apache.hadoop.ozone.om.exceptions.OMException.ResultCodes.KEY_NOT_FOUND;
 import static 
org.apache.hadoop.ozone.om.exceptions.OMException.ResultCodes.NOT_SUPPORTED_OPERATION_PRIOR_FINALIZATION;
 import static 
org.apache.hadoop.ozone.om.helpers.BucketLayout.FILE_SYSTEM_OPTIMIZED;
@@ -108,6 +109,7 @@ import static org.awaitility.Awaitility.await;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertNotNull;
 import static org.junit.Assert.assertNull;
+import static org.junit.Assert.assertTrue;
 import static org.junit.Assert.fail;
 import static org.junit.Assert.assertThrows;
 import static java.nio.charset.StandardCharsets.UTF_8;
@@ -225,7 +227,7 @@ public class TestOmSnapshot {
 
   private static void expectFailurePreFinalization(LambdaTestUtils.
       VoidCallable eval) throws Exception {
-    OMException ex  = Assert.assertThrows(OMException.class,
+    OMException ex  = assertThrows(OMException.class,
             () -> eval.call());
     Assert.assertEquals(ex.getResult(),
             NOT_SUPPORTED_OPERATION_PRIOR_FINALIZATION);
@@ -583,6 +585,12 @@ public class TestOmSnapshot {
     key1 = createFileKeyWithPrefix(bucket1, key1);
     String snap1 = "snap" + counter.incrementAndGet();
     createSnapshot(volume, bucket, snap1);
+
+    // When from and to snapshots are same, it returns empty response.
+    SnapshotDiffReportOzone
+        diff0 = getSnapDiffReport(volume, bucket, snap1, snap1);
+    assertTrue(diff0.getDiffList().isEmpty());
+
     // Do nothing, take another snapshot
     String snap2 = "snap" + counter.incrementAndGet();
     createSnapshot(volume, bucket, snap2);
@@ -809,16 +817,28 @@ public class TestOmSnapshot {
     String snap1 = "snap" + counter.incrementAndGet();
     createSnapshot(volume, bucket, snap1);
     String snap2 = "snap" + counter.incrementAndGet();
+
     // Destination snapshot is invalid
-    LambdaTestUtils.intercept(OMException.class,
-            "KEY_NOT_FOUND",
+    OMException omException = assertThrows(OMException.class,
             () -> store.snapshotDiff(volume, bucket, snap1, snap2,
                 null, 0, false, false));
+    assertEquals(KEY_NOT_FOUND, omException.getResult());
     // From snapshot is invalid
-    LambdaTestUtils.intercept(OMException.class,
-            "KEY_NOT_FOUND",
-            () -> store.snapshotDiff(volume, bucket, snap2, snap1,
-                null, 0, false, false));
+    omException = assertThrows(OMException.class,
+        () -> store.snapshotDiff(volume, bucket, snap2, snap1,
+            null, 0, false, false));
+
+    assertEquals(KEY_NOT_FOUND, omException.getResult());
+
+    createSnapshot(volume, bucket, snap2);
+
+    omException = assertThrows(OMException.class, () ->
+        store.snapshotDiff(volume, bucket, snap2, snap1, null, 0, false, false)
+    );
+
+    assertEquals(INTERNAL_ERROR, omException.getResult());
+    assertEquals("fromSnapshot:" + snap2 + " should be older than to " +
+        "toSnapshot:" + snap1, omException.getMessage());
   }
 
   @Test
diff --git 
a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OmSnapshotManager.java
 
b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OmSnapshotManager.java
index c2d662eed5..c1fdbc0172 100644
--- 
a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OmSnapshotManager.java
+++ 
b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OmSnapshotManager.java
@@ -32,6 +32,7 @@ import java.nio.file.Path;
 import java.nio.file.Paths;
 import java.util.ArrayList;
 import java.util.Arrays;
+import java.util.Collections;
 import java.util.HashSet;
 import java.util.List;
 import java.util.Objects;
@@ -96,8 +97,10 @@ import static 
org.apache.hadoop.ozone.om.OMConfigKeys.OZONE_OM_SNAPSHOT_DIFF_DB_
 import static 
org.apache.hadoop.ozone.om.OMConfigKeys.OZONE_OM_SNAPSHOT_DIFF_REPORT_MAX_PAGE_SIZE;
 import static 
org.apache.hadoop.ozone.om.OMConfigKeys.OZONE_OM_SNAPSHOT_DIFF_REPORT_MAX_PAGE_SIZE_DEFAULT;
 import static 
org.apache.hadoop.ozone.om.exceptions.OMException.ResultCodes.INVALID_KEY_NAME;
+import static 
org.apache.hadoop.ozone.om.snapshot.SnapshotDiffManager.getSnapshotRootPath;
 import static 
org.apache.hadoop.ozone.om.snapshot.SnapshotUtils.checkSnapshotActive;
 import static 
org.apache.hadoop.ozone.om.snapshot.SnapshotUtils.dropColumnFamilyHandle;
+import static 
org.apache.hadoop.ozone.snapshot.SnapshotDiffResponse.JobStatus.DONE;
 
 /**
  * This class is used to manage/create OM snapshots.
@@ -711,6 +714,14 @@ public final class OmSnapshotManager implements 
AutoCloseable {
 
     validateSnapshotsExistAndActive(volume, bucket, fromSnapshot, toSnapshot);
 
+    // Check if fromSnapshot and toSnapshot are equal.
+    if (Objects.equals(fromSnapshot, toSnapshot)) {
+      SnapshotDiffReportOzone diffReport = new SnapshotDiffReportOzone(
+          getSnapshotRootPath(volume, bucket).toString(), volume, bucket,
+          fromSnapshot, toSnapshot, Collections.emptyList(), null);
+      return new SnapshotDiffResponse(diffReport, DONE, 0L);
+    }
+
     int index = getIndexFromToken(token);
     if (pageSize <= 0 || pageSize > maxPageSize) {
       pageSize = maxPageSize;
@@ -776,6 +787,7 @@ public final class OmSnapshotManager implements 
AutoCloseable {
     // Block SnapDiff if either of the snapshots is not active.
     checkSnapshotActive(fromSnapInfo, false);
     checkSnapshotActive(toSnapInfo, false);
+
     // Check snapshot creation time
     if (fromSnapInfo.getCreationTime() > toSnapInfo.getCreationTime()) {
       throw new IOException("fromSnapshot:" + fromSnapInfo.getName() +
diff --git 
a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/snapshot/SnapshotDiffManager.java
 
b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/snapshot/SnapshotDiffManager.java
index ac70c44d56..0905a5c3b7 100644
--- 
a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/snapshot/SnapshotDiffManager.java
+++ 
b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/snapshot/SnapshotDiffManager.java
@@ -555,7 +555,7 @@ public class SnapshotDiffManager implements AutoCloseable {
   }
 
   @NotNull
-  private static OFSPath getSnapshotRootPath(String volume, String bucket) {
+  public static OFSPath getSnapshotRootPath(String volume, String bucket) {
     org.apache.hadoop.fs.Path bucketPath = new org.apache.hadoop.fs.Path(
         OZONE_URI_DELIMITER + volume + OZONE_URI_DELIMITER + bucket);
     return new OFSPath(bucketPath, new OzoneConfiguration());
@@ -1200,9 +1200,14 @@ public class SnapshotDiffManager implements 
AutoCloseable {
           getDSIFromSI(tsInfo, toSnapshot, volume, bucket);
 
       LOG.debug("Calling RocksDBCheckpointDiffer");
-      List<String> sstDiffList =
-          differ.getSSTDiffListWithFullPath(toDSI, fromDSI, diffDir);
-      deltaFiles.addAll(sstDiffList);
+      try {
+        List<String> sstDiffList =
+            differ.getSSTDiffListWithFullPath(toDSI, fromDSI, diffDir);
+        deltaFiles.addAll(sstDiffList);
+      } catch (Exception exception) {
+        LOG.warn("Failed to get SST diff file using RocksDBCheckpointDiffer. " 
+
+            "It will fallback to full diff now.", exception);
+      }
     }
 
     if (useFullDiff || deltaFiles.isEmpty()) {
diff --git 
a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/snapshot/TestSnapshotDiffManager.java
 
b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/snapshot/TestSnapshotDiffManager.java
index 46f1773f7b..6e50a6fa79 100644
--- 
a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/snapshot/TestSnapshotDiffManager.java
+++ 
b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/snapshot/TestSnapshotDiffManager.java
@@ -96,6 +96,7 @@ import org.rocksdb.RocksDBException;
 import org.rocksdb.RocksIterator;
 
 import java.io.File;
+import java.io.FileNotFoundException;
 import java.io.IOException;
 import java.nio.file.Files;
 import java.util.ArrayList;
@@ -163,6 +164,7 @@ import static org.mockito.Mockito.any;
 import static org.mockito.Mockito.doAnswer;
 import static org.mockito.Mockito.doNothing;
 import static org.mockito.Mockito.doReturn;
+import static org.mockito.Mockito.doThrow;
 import static org.mockito.Mockito.mock;
 import static org.mockito.Mockito.spy;
 import static org.mockito.Mockito.when;
@@ -459,22 +461,78 @@ public class TestSnapshotDiffManager {
                 });
             return null;
           });
-      SnapshotDiffManager spy = spy(snapshotDiffManager);
       UUID snap1 = UUID.randomUUID();
       UUID snap2 = UUID.randomUUID();
       if (!useFullDiff) {
-        Set<String> randomStrings = Collections.emptySet();
         when(differ.getSSTDiffListWithFullPath(
             any(DifferSnapshotInfo.class),
             any(DifferSnapshotInfo.class),
             anyString()))
-            .thenReturn(Lists.newArrayList(randomStrings));
+            .thenReturn(Collections.emptyList());
       }
 
       SnapshotInfo fromSnapshotInfo = getMockedSnapshotInfo(snap1);
       SnapshotInfo toSnapshotInfo = getMockedSnapshotInfo(snap1);
       when(jobTableIterator.isValid()).thenReturn(false);
-      Set<String> deltaFiles = spy.getDeltaFiles(
+      Set<String> deltaFiles = snapshotDiffManager.getDeltaFiles(
+          snapshotCache.get(snap1.toString()),
+          snapshotCache.get(snap2.toString()),
+          Arrays.asList("cf1", "cf2"),
+          fromSnapshotInfo,
+          toSnapshotInfo,
+          false,
+          Collections.emptyMap(),
+          Files.createTempDirectory("snapdiff_dir").toAbsolutePath()
+              .toString());
+      assertEquals(deltaStrings, deltaFiles);
+    }
+  }
+
+  @ParameterizedTest
+  @ValueSource(ints = {0, 1, 2, 5, 10, 100, 1000, 10000})
+  public void testGetDeltaFilesWithDifferThrowException(int numberOfFiles)
+      throws ExecutionException, RocksDBException, IOException {
+    try (MockedStatic<RdbUtil> mockedRdbUtil =
+             Mockito.mockStatic(RdbUtil.class);
+         MockedStatic<RocksDiffUtils> mockedRocksDiffUtils =
+             Mockito.mockStatic(RocksDiffUtils.class)) {
+      Set<String> deltaStrings = new HashSet<>();
+
+      mockedRdbUtil.when(
+              () -> RdbUtil.getSSTFilesForComparison(anyString(), anyList()))
+          .thenAnswer((Answer<Set<String>>) invocation -> {
+            Set<String> retVal = IntStream.range(0, numberOfFiles)
+                .mapToObj(i -> RandomStringUtils.randomAlphabetic(10))
+                .collect(Collectors.toSet());
+            deltaStrings.addAll(retVal);
+            return retVal;
+          });
+
+      mockedRocksDiffUtils.when(() ->
+              RocksDiffUtils.filterRelevantSstFiles(anySet(), anyMap()))
+          .thenAnswer((Answer<Void>) invocationOnMock -> {
+            invocationOnMock.getArgument(0, Set.class).stream()
+                .findAny().ifPresent(val -> {
+                  assertTrue(deltaStrings.contains(val));
+                  invocationOnMock.getArgument(0, Set.class).remove(val);
+                  deltaStrings.remove(val);
+                });
+            return null;
+          });
+      UUID snap1 = UUID.randomUUID();
+      UUID snap2 = UUID.randomUUID();
+
+      doThrow(new FileNotFoundException("File not found exception."))
+          .when(differ)
+          .getSSTDiffListWithFullPath(
+              any(DifferSnapshotInfo.class),
+              any(DifferSnapshotInfo.class),
+              anyString());
+
+      SnapshotInfo fromSnapshotInfo = getMockedSnapshotInfo(snap1);
+      SnapshotInfo toSnapshotInfo = getMockedSnapshotInfo(snap1);
+      when(jobTableIterator.isValid()).thenReturn(false);
+      Set<String> deltaFiles = snapshotDiffManager.getDeltaFiles(
           snapshotCache.get(snap1.toString()),
           snapshotCache.get(snap2.toString()),
           Arrays.asList("cf1", "cf2"),


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to