You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ozone.apache.org by sa...@apache.org on 2020/07/24 02:17:55 UTC

[hadoop-ozone] branch master updated: HDDS-3658. Stop to persist container related pipeline info of each ke… (#1012)

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

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


The following commit(s) were added to refs/heads/master by this push:
     new 71cc33d  HDDS-3658. Stop to persist container related pipeline info of each ke… (#1012)
71cc33d is described below

commit 71cc33da2747b0e68bc4157b45d23933b047950a
Author: Sammi Chen <sa...@apache.org>
AuthorDate: Fri Jul 24 10:17:44 2020 +0800

    HDDS-3658. Stop to persist container related pipeline info of each ke… (#1012)
---
 .../java/org/apache/hadoop/hdds/scm/TestUtils.java |  13 +++
 .../apache/hadoop/ozone/om/helpers/OmKeyInfo.java  |  17 ++-
 .../hadoop/ozone/om/helpers/OmKeyLocationInfo.java |  18 ++-
 .../ozone/om/helpers/OmKeyLocationInfoGroup.java   |  22 ++--
 .../hadoop/ozone/om/helpers/RepeatedOmKeyInfo.java |   9 +-
 .../apache/hadoop/ozone/om/TestKeyManagerImpl.java |  14 ++-
 .../ozone/recon/TestReconWithOzoneManager.java     |  14 +--
 .../org/apache/hadoop/ozone/om/KeyManager.java     |   6 +
 .../org/apache/hadoop/ozone/om/KeyManagerImpl.java | 115 ++++++++++++++------
 .../hadoop/ozone/om/OmMetadataManagerImpl.java     |   9 +-
 .../hadoop/ozone/om/codec/OMDBDefinition.java      |   6 +-
 .../hadoop/ozone/om/codec/OmKeyInfoCodec.java      |  13 ++-
 .../ozone/om/codec/RepeatedOmKeyInfoCodec.java     |  13 ++-
 .../ozone/om/request/file/OMFileCreateRequest.java |   4 +
 .../ozone/om/request/key/OMKeyCreateRequest.java   |   4 +
 .../protocolPB/OzoneManagerRequestHandler.java     |   2 +-
 .../apache/hadoop/ozone/om/TestKeyManagerUnit.java |  20 ----
 .../hadoop/ozone/om/codec/TestOmKeyInfoCodec.java  | 116 ++++++++++++++++++++
 .../ozone/om/codec/TestRepeatedOmKeyInfoCodec.java | 121 +++++++++++++++++++++
 .../ozone/om/request/key/TestOMKeyRequest.java     |   6 +
 .../apache/hadoop/ozone/fsck/ContainerMapper.java  |   2 +-
 21 files changed, 453 insertions(+), 91 deletions(-)

diff --git a/hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestUtils.java b/hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestUtils.java
index 64752da..03ed0f7 100644
--- a/hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestUtils.java
+++ b/hadoop-hdds/server-scm/src/test/java/org/apache/hadoop/hdds/scm/TestUtils.java
@@ -547,4 +547,17 @@ public final class TestUtils {
         .build();
   }
 
+  public static Pipeline getRandomPipeline() {
+    List<DatanodeDetails> nodes = new ArrayList<>();
+    nodes.add(MockDatanodeDetails.randomDatanodeDetails());
+    nodes.add(MockDatanodeDetails.randomDatanodeDetails());
+    nodes.add(MockDatanodeDetails.randomDatanodeDetails());
+    return Pipeline.newBuilder()
+        .setFactor(HddsProtos.ReplicationFactor.THREE)
+        .setId(PipelineID.randomId())
+        .setNodes(nodes)
+        .setState(Pipeline.PipelineState.OPEN)
+        .setType(HddsProtos.ReplicationType.RATIS)
+        .build();
+  }
 }
diff --git a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyInfo.java b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyInfo.java
index 6304e39..d0e8bee 100644
--- a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyInfo.java
+++ b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyInfo.java
@@ -377,13 +377,26 @@ public final class OmKeyInfo extends WithObjectID {
     }
   }
 
+  /**
+   * For network transmit.
+   * @return
+   */
   public KeyInfo getProtobuf() {
+    return getProtobuf(false);
+  }
+
+  /**
+   *
+   * @param ignorePipeline true for persist to DB, false for network transmit.
+   * @return
+   */
+  public KeyInfo getProtobuf(boolean ignorePipeline) {
     long latestVersion = keyLocationVersions.size() == 0 ? -1 :
         keyLocationVersions.get(keyLocationVersions.size() - 1).getVersion();
 
     List<KeyLocationList> keyLocations = new ArrayList<>();
     for (OmKeyLocationInfoGroup locationInfoGroup : keyLocationVersions) {
-      keyLocations.add(locationInfoGroup.getProtobuf());
+      keyLocations.add(locationInfoGroup.getProtobuf(ignorePipeline));
     }
 
     KeyInfo.Builder kb = KeyInfo.newBuilder()
@@ -393,8 +406,8 @@ public final class OmKeyInfo extends WithObjectID {
         .setDataSize(dataSize)
         .setFactor(factor)
         .setType(type)
-        .addAllKeyLocationList(keyLocations)
         .setLatestVersion(latestVersion)
+        .addAllKeyLocationList(keyLocations)
         .setCreationTime(creationTime)
         .setModificationTime(modificationTime)
         .addAllMetadata(KeyValueUtil.toProtobuf(metadata))
diff --git a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyLocationInfo.java b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyLocationInfo.java
index b9a2920..70c71d6 100644
--- a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyLocationInfo.java
+++ b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyLocationInfo.java
@@ -156,7 +156,15 @@ public final class OmKeyLocationInfo {
     }
   }
 
+  public KeyLocation getCompactProtobuf() {
+    return getProtobuf(true);
+  }
+
   public KeyLocation getProtobuf() {
+    return getProtobuf(false);
+  }
+
+  private KeyLocation getProtobuf(boolean ignorePipeline) {
     KeyLocation.Builder builder = KeyLocation.newBuilder()
         .setBlockID(blockID.getProtobuf())
         .setLength(length)
@@ -165,10 +173,12 @@ public final class OmKeyLocationInfo {
     if (this.token != null) {
       builder.setToken(OzonePBHelper.protoFromToken(token));
     }
-    try {
-      builder.setPipeline(pipeline.getProtobufMessage());
-    } catch (UnknownPipelineStateException e) {
-      //TODO: fix me: we should not return KeyLocation without pipeline.
+    if (!ignorePipeline) {
+      try {
+        builder.setPipeline(pipeline.getProtobufMessage());
+      } catch (UnknownPipelineStateException e) {
+        //TODO: fix me: we should not return KeyLocation without pipeline.
+      }
     }
     return builder.build();
   }
diff --git a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyLocationInfoGroup.java b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyLocationInfoGroup.java
index edebb67..abaf055 100644
--- a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyLocationInfoGroup.java
+++ b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/OmKeyLocationInfoGroup.java
@@ -16,6 +16,7 @@
  */
 package org.apache.hadoop.ozone.om.helpers;
 
+import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos;
 import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.KeyLocationList;
 
 import java.util.ArrayList;
@@ -75,15 +76,18 @@ public class OmKeyLocationInfoGroup {
     return new ArrayList<>(locationVersionMap.get(versionToFetch));
   }
 
-  public KeyLocationList getProtobuf() {
-    return KeyLocationList.newBuilder()
-        .setVersion(version)
-        .addAllKeyLocations(
-            locationVersionMap.values().stream()
-                .flatMap(List::stream)
-                .map(OmKeyLocationInfo::getProtobuf)
-                .collect(Collectors.toList()))
-        .build();
+  public KeyLocationList getProtobuf(boolean ignorePipeline) {
+    KeyLocationList.Builder builder = KeyLocationList.newBuilder()
+        .setVersion(version);
+    List<OzoneManagerProtocolProtos.KeyLocation> keyLocationList =
+        new ArrayList<>();
+    for (List<OmKeyLocationInfo> locationList : locationVersionMap.values()) {
+      for (OmKeyLocationInfo keyInfo : locationList) {
+        keyLocationList.add(ignorePipeline ?
+            keyInfo.getCompactProtobuf() : keyInfo.getProtobuf());
+      }
+    }
+    return  builder.addAllKeyLocations(keyLocationList).build();
   }
 
   public static OmKeyLocationInfoGroup getFromProtobuf(
diff --git a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/RepeatedOmKeyInfo.java b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/RepeatedOmKeyInfo.java
index c28c2c8..6d7bf2f 100644
--- a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/RepeatedOmKeyInfo.java
+++ b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/helpers/RepeatedOmKeyInfo.java
@@ -60,10 +60,15 @@ public class RepeatedOmKeyInfo {
     return new RepeatedOmKeyInfo.Builder().setOmKeyInfos(list).build();
   }
 
-  public RepeatedKeyInfo getProto() {
+  /**
+   *
+   * @param compact, true for persistence, false for network transmit
+   * @return
+   */
+  public RepeatedKeyInfo getProto(boolean compact) {
     List<KeyInfo> list = new ArrayList<>();
     for(OmKeyInfo k : omKeyInfoList) {
-      list.add(k.getProtobuf());
+      list.add(k.getProtobuf(compact));
     }
 
     RepeatedKeyInfo.Builder builder = RepeatedKeyInfo.newBuilder()
diff --git a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/om/TestKeyManagerImpl.java b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/om/TestKeyManagerImpl.java
index 7a1cb5b..a2b70a5 100644
--- a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/om/TestKeyManagerImpl.java
+++ b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/om/TestKeyManagerImpl.java
@@ -23,6 +23,7 @@ import java.io.IOException;
 import java.nio.file.Path;
 import java.nio.file.Paths;
 import java.util.ArrayList;
+import java.util.Arrays;
 import java.util.BitSet;
 import java.util.Collections;
 import java.util.HashMap;
@@ -137,6 +138,7 @@ public class TestKeyManagerImpl {
   private static NodeManager nodeManager;
   private static StorageContainerManager scm;
   private static ScmBlockLocationProtocol mockScmBlockLocationProtocol;
+  private static StorageContainerLocationProtocol mockScmContainerClient;
   private static OzoneConfiguration conf;
   private static OMMetadataManager metadataManager;
   private static File dir;
@@ -178,9 +180,11 @@ public class TestKeyManagerImpl {
             StorageUnit.BYTES);
     conf.setLong(OZONE_KEY_PREALLOCATION_BLOCKS_MAX, 10);
 
+    mockScmContainerClient =
+        Mockito.mock(StorageContainerLocationProtocol.class);
     keyManager =
-        new KeyManagerImpl(scm.getBlockProtocolServer(), metadataManager, conf,
-            "om1", null);
+        new KeyManagerImpl(scm.getBlockProtocolServer(),
+            mockScmContainerClient, metadataManager, conf, "om1", null);
     prefixManager = new PrefixManagerImpl(metadataManager, false);
 
     Mockito.when(mockScmBlockLocationProtocol
@@ -763,6 +767,12 @@ public class TestKeyManagerImpl {
     keyArgs.setLocationInfoList(locationInfoList);
 
     keyManager.commitKey(keyArgs, keySession.getId());
+    ContainerInfo containerInfo = new ContainerInfo.Builder().setContainerID(1L)
+        .setPipelineID(pipeline.getId()).build();
+    List<ContainerWithPipeline> containerWithPipelines = Arrays.asList(
+        new ContainerWithPipeline(containerInfo, pipeline));
+    when(mockScmContainerClient.getContainerWithPipelineBatch(
+        Arrays.asList(1L))).thenReturn(containerWithPipelines);
 
     OmKeyInfo key = keyManager.lookupKey(keyArgs, null);
     Assert.assertEquals(key.getKeyName(), keyName);
diff --git a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconWithOzoneManager.java b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconWithOzoneManager.java
index 9363054..af92001 100644
--- a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconWithOzoneManager.java
+++ b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconWithOzoneManager.java
@@ -40,8 +40,8 @@ import java.util.concurrent.TimeUnit;
 import org.apache.hadoop.hdds.client.BlockID;
 import org.apache.hadoop.hdds.conf.OzoneConfiguration;
 import org.apache.hadoop.hdds.protocol.proto.HddsProtos;
+import org.apache.hadoop.hdds.scm.TestUtils;
 import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
-import org.apache.hadoop.hdds.scm.pipeline.PipelineID;
 import org.apache.hadoop.hdds.utils.db.RDBStore;
 import org.apache.hadoop.hdds.utils.db.Table;
 import org.apache.hadoop.hdds.utils.db.TableIterator;
@@ -335,7 +335,7 @@ public class TestReconWithOzoneManager {
    */
   private void addKeys(int start, int end) throws Exception {
     for(int i = start; i < end; i++) {
-      Pipeline pipeline = getRandomPipeline();
+      Pipeline pipeline = TestUtils.getRandomPipeline();
       List<OmKeyLocationInfo> omKeyLocationInfoList = new ArrayList<>();
       BlockID blockID = new BlockID(i, 1);
       OmKeyLocationInfo omKeyLocationInfo1 = getOmKeyLocationInfo(blockID,
@@ -358,16 +358,6 @@ public class TestReconWithOzoneManager {
     return keyCount;
   }
 
-  private static Pipeline getRandomPipeline() {
-    return Pipeline.newBuilder()
-        .setFactor(HddsProtos.ReplicationFactor.ONE)
-        .setId(PipelineID.randomId())
-        .setNodes(Collections.EMPTY_LIST)
-        .setState(Pipeline.PipelineState.OPEN)
-        .setType(HddsProtos.ReplicationType.STAND_ALONE)
-        .build();
-  }
-
   private static OmKeyLocationInfo getOmKeyLocationInfo(BlockID blockID,
       Pipeline pipeline) {
     return new OmKeyLocationInfo.Builder()
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/KeyManager.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/KeyManager.java
index 2088f5d..dbcecc8 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/KeyManager.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/KeyManager.java
@@ -266,4 +266,10 @@ public interface KeyManager extends OzoneManagerFS, IOzoneAcl {
   OmMultipartUploadListParts listParts(String volumeName, String bucketName,
       String keyName, String uploadID, int partNumberMarker,
       int maxParts)  throws IOException;
+
+  /**
+   * Refresh the key block location information by get latest info from SCM.
+   * @param key
+   */
+  void refresh(OmKeyInfo key) throws IOException;
 }
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/KeyManagerImpl.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/KeyManagerImpl.java
index 0feba7c..66ddeb6 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/KeyManagerImpl.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/KeyManagerImpl.java
@@ -23,6 +23,7 @@ import java.security.GeneralSecurityException;
 import java.security.PrivilegedExceptionAction;
 import java.time.Instant;
 import java.util.ArrayList;
+import java.util.Arrays;
 import java.util.Collections;
 import java.util.EnumSet;
 import java.util.HashMap;
@@ -54,6 +55,7 @@ import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline
 import org.apache.hadoop.hdds.scm.container.common.helpers.ExcludeList;
 import org.apache.hadoop.hdds.scm.exceptions.SCMException;
 import org.apache.hadoop.hdds.scm.protocol.ScmBlockLocationProtocol;
+import org.apache.hadoop.hdds.scm.protocol.StorageContainerLocationProtocol;
 import org.apache.hadoop.hdds.utils.BackgroundService;
 import org.apache.hadoop.hdds.utils.UniqueId;
 import org.apache.hadoop.hdds.utils.db.BatchOperation;
@@ -105,7 +107,6 @@ import com.google.common.annotations.VisibleForTesting;
 import com.google.common.base.Preconditions;
 import com.google.common.base.Strings;
 import org.apache.commons.codec.digest.DigestUtils;
-import org.apache.commons.collections.CollectionUtils;
 import org.apache.commons.lang3.StringUtils;
 import static org.apache.hadoop.fs.CommonConfigurationKeysPublic.HADOOP_SECURITY_KEY_PROVIDER_PATH;
 import static org.apache.hadoop.hdds.HddsConfigKeys.HDDS_BLOCK_TOKEN_ENABLED;
@@ -173,6 +174,15 @@ public class KeyManagerImpl implements KeyManager {
         conf, omId, secretManager, null, null);
   }
 
+  @VisibleForTesting
+  public KeyManagerImpl(ScmBlockLocationProtocol scmBlockClient,
+      StorageContainerLocationProtocol scmContainerClient,
+      OMMetadataManager metadataManager, OzoneConfiguration conf, String omId,
+      OzoneBlockTokenSecretManager secretManager) {
+    this(null, new ScmClient(scmBlockClient, scmContainerClient),
+        metadataManager, conf, omId, secretManager, null, null);
+  }
+
   public KeyManagerImpl(OzoneManager om, ScmClient scmClient,
       OzoneConfiguration conf, String omId) {
     this (om, scmClient, om.getMetadataManager(), conf, omId,
@@ -678,9 +688,8 @@ public class KeyManagerImpl implements KeyManager {
     // Refresh container pipeline info from SCM
     // based on OmKeyArgs.refreshPipeline flag
     // value won't be null as the check is done inside try/catch block.
-    if (args.getRefreshPipeline()) {
-      refreshPipeline(value);
-    }
+    refreshPipeline(value);
+
     if (args.getSortDatanodes()) {
       sortDatanodeInPipeline(value, clientAddress);
     }
@@ -693,23 +702,64 @@ public class KeyManagerImpl implements KeyManager {
    */
   @VisibleForTesting
   protected void refreshPipeline(OmKeyInfo value) throws IOException {
-    final List<OmKeyLocationInfoGroup> locationInfoGroups = value == null ?
-        null : value.getKeyLocationVersions();
+    Preconditions.checkNotNull(value, "OMKeyInfo cannot be null");
+    refreshPipeline(Arrays.asList(value));
+  }
 
-    // TODO: fix Some tests that may not initialize container client
-    // The production should always have containerClient initialized.
-    if (scmClient.getContainerClient() == null ||
-        CollectionUtils.isEmpty(locationInfoGroups)) {
+  /**
+   * Refresh pipeline info in OM by asking SCM.
+   * @param keyList a list of OmKeyInfo
+   */
+  @VisibleForTesting
+  protected void refreshPipeline(List<OmKeyInfo> keyList) throws IOException {
+    if (keyList == null || keyList.isEmpty()) {
       return;
     }
 
     Set<Long> containerIDs = new HashSet<>();
-    for (OmKeyLocationInfoGroup key : locationInfoGroups) {
-      for (OmKeyLocationInfo k : key.getLocationList()) {
-        containerIDs.add(k.getContainerID());
+    for (OmKeyInfo keyInfo : keyList) {
+      List<OmKeyLocationInfoGroup> locationInfoGroups =
+          keyInfo.getKeyLocationVersions();
+
+      for (OmKeyLocationInfoGroup key : locationInfoGroups) {
+        for (OmKeyLocationInfo k : key.getLocationList()) {
+          containerIDs.add(k.getContainerID());
+        }
       }
     }
 
+    Map<Long, ContainerWithPipeline> containerWithPipelineMap =
+        refreshPipeline(containerIDs);
+
+    for (OmKeyInfo keyInfo : keyList) {
+      List<OmKeyLocationInfoGroup> locationInfoGroups =
+          keyInfo.getKeyLocationVersions();
+      for (OmKeyLocationInfoGroup key : locationInfoGroups) {
+        for (OmKeyLocationInfo k : key.getLocationList()) {
+          ContainerWithPipeline cp =
+              containerWithPipelineMap.get(k.getContainerID());
+          if (cp != null && !cp.getPipeline().equals(k.getPipeline())) {
+            k.setPipeline(cp.getPipeline());
+          }
+        }
+      }
+    }
+  }
+
+  /**
+   * Refresh pipeline info in OM by asking SCM.
+   * @param containerIDs a set of containerIDs
+   */
+  @VisibleForTesting
+  protected Map<Long, ContainerWithPipeline> refreshPipeline(
+      Set<Long> containerIDs) throws IOException {
+    // TODO: fix Some tests that may not initialize container client
+    // The production should always have containerClient initialized.
+    if (scmClient.getContainerClient() == null ||
+        containerIDs == null || containerIDs.isEmpty()) {
+      return Collections.EMPTY_MAP;
+    }
+
     Map<Long, ContainerWithPipeline> containerWithPipelineMap = new HashMap<>();
 
     try {
@@ -719,22 +769,12 @@ public class KeyManagerImpl implements KeyManager {
         containerWithPipelineMap.put(
             cp.getContainerInfo().getContainerID(), cp);
       }
+      return containerWithPipelineMap;
     } catch (IOException ioEx) {
-      LOG.debug("Get containerPipeline failed for volume:{} bucket:{} " +
-          "key:{}", value.getVolumeName(), value.getBucketName(),
-          value.getKeyName(), ioEx);
+      LOG.debug("Get containerPipeline failed for {}",
+          containerIDs.toString(), ioEx);
       throw new OMException(ioEx.getMessage(), SCM_GET_PIPELINE_EXCEPTION);
     }
-
-    for (OmKeyLocationInfoGroup key : locationInfoGroups) {
-      for (OmKeyLocationInfo k : key.getLocationList()) {
-        ContainerWithPipeline cp =
-            containerWithPipelineMap.get(k.getContainerID());
-        if (!cp.getPipeline().equals(k.getPipeline())) {
-          k.setPipeline(cp.getPipeline());
-        }
-      }
-    }
   }
 
   @Override
@@ -872,8 +912,10 @@ public class KeyManagerImpl implements KeyManager {
     // underlying table using an iterator. That automatically creates a
     // snapshot of the data, so we don't need these locks at a higher level
     // when we iterate.
-    return metadataManager.listKeys(volumeName, bucketName,
+    List<OmKeyInfo> keyList = metadataManager.listKeys(volumeName, bucketName,
         startKey, keyPrefix, maxKeys);
+    refreshPipeline(keyList);
+    return keyList;
   }
 
   @Override
@@ -1742,9 +1784,7 @@ public class KeyManagerImpl implements KeyManager {
 
       // if the key is a file then do refresh pipeline info in OM by asking SCM
       if (fileKeyInfo != null) {
-        if (refreshPipeline) {
-          refreshPipeline(fileKeyInfo);
-        }
+        refreshPipeline(fileKeyInfo);
         if (sortDatanodes) {
           sortDatanodeInPipeline(fileKeyInfo, clientAddress);
         }
@@ -1914,6 +1954,15 @@ public class KeyManagerImpl implements KeyManager {
   }
 
   /**
+   * Refresh the key block location information by get latest info from SCM.
+   * @param key
+   */
+  public void refresh(OmKeyInfo key) throws IOException {
+    Preconditions.checkNotNull(key, "Key info can not be null");
+    refreshPipeline(Arrays.asList(key));
+  }
+
+  /**
    * Helper function for listStatus to find key in TableCache.
    */
   private void listStatusFindKeyInTableCache(
@@ -2079,7 +2128,11 @@ public class KeyManagerImpl implements KeyManager {
       for (Map.Entry<String, OzoneFileStatus> entry : cacheKeyMap.entrySet()) {
         // No need to check if a key is deleted or not here, this is handled
         // when adding entries to cacheKeyMap from DB.
-        fileStatusList.add(entry.getValue());
+        OzoneFileStatus fileStatus = entry.getValue();
+        if (fileStatus.isFile()) {
+          refreshPipeline(fileStatus.getKeyInfo());
+        }
+        fileStatusList.add(fileStatus);
         countEntries++;
         if (countEntries >= numEntries) {
           break;
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OmMetadataManagerImpl.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OmMetadataManagerImpl.java
index 6c8b505..e64b023 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OmMetadataManagerImpl.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OmMetadataManagerImpl.java
@@ -157,6 +157,7 @@ public class OmMetadataManagerImpl implements OMMetadataManager {
   private Table prefixTable;
   private Table transactionInfoTable;
   private boolean isRatisEnabled;
+  private boolean ignorePipelineinKey;
 
   private Map<String, Table> tableMap = new HashMap<>();
 
@@ -172,6 +173,9 @@ public class OmMetadataManagerImpl implements OMMetadataManager {
     isRatisEnabled = conf.getBoolean(
         OMConfigKeys.OZONE_OM_RATIS_ENABLE_KEY,
         OMConfigKeys.OZONE_OM_RATIS_ENABLE_DEFAULT);
+    // For test purpose only
+    ignorePipelineinKey = conf.getBoolean(
+        "ozone.om.ignore.pipeline", Boolean.TRUE);
     start(conf);
   }
 
@@ -317,8 +321,9 @@ public class OmMetadataManagerImpl implements OMMetadataManager {
         .addTable(PREFIX_TABLE)
         .addTable(TRANSACTION_INFO_TABLE)
         .addCodec(OzoneTokenIdentifier.class, new TokenIdentifierCodec())
-        .addCodec(OmKeyInfo.class, new OmKeyInfoCodec())
-        .addCodec(RepeatedOmKeyInfo.class, new RepeatedOmKeyInfoCodec())
+        .addCodec(OmKeyInfo.class, new OmKeyInfoCodec(true))
+        .addCodec(RepeatedOmKeyInfo.class,
+            new RepeatedOmKeyInfoCodec(true))
         .addCodec(OmBucketInfo.class, new OmBucketInfoCodec())
         .addCodec(OmVolumeArgs.class, new OmVolumeArgsCodec())
         .addCodec(UserVolumeInfo.class, new UserVolumeInfoCodec())
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/OMDBDefinition.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/OMDBDefinition.java
index 24f3341..f6d04a9 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/OMDBDefinition.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/OMDBDefinition.java
@@ -48,7 +48,7 @@ public class OMDBDefinition implements DBDefinition {
                     String.class,
                     new StringCodec(),
                     RepeatedOmKeyInfo.class,
-                    new RepeatedOmKeyInfoCodec());
+                    new RepeatedOmKeyInfoCodec(true));
 
   public static final DBColumnFamilyDefinition<String,
             OzoneManagerProtocolProtos.UserVolumeInfo>
@@ -85,7 +85,7 @@ public class OMDBDefinition implements DBDefinition {
                     String.class,
                     new StringCodec(),
                     OmKeyInfo.class,
-                    new OmKeyInfoCodec());
+                    new OmKeyInfoCodec(true));
 
   public static final DBColumnFamilyDefinition<String, OmKeyInfo>
             KEY_TABLE =
@@ -94,7 +94,7 @@ public class OMDBDefinition implements DBDefinition {
                     String.class,
                     new StringCodec(),
                     OmKeyInfo.class,
-                    new OmKeyInfoCodec());
+                    new OmKeyInfoCodec(true));
 
   public static final DBColumnFamilyDefinition<String, OmBucketInfo>
             BUCKET_TABLE =
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/OmKeyInfoCodec.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/OmKeyInfoCodec.java
index c6b86bb..a7e1eab 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/OmKeyInfoCodec.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/OmKeyInfoCodec.java
@@ -18,23 +18,34 @@
 package org.apache.hadoop.ozone.om.codec;
 
 import java.io.IOException;
+
 import org.apache.hadoop.ozone.om.helpers.OmKeyInfo;
 import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.KeyInfo;
 import org.apache.hadoop.hdds.utils.db.Codec;
 
 import com.google.common.base.Preconditions;
 import com.google.protobuf.InvalidProtocolBufferException;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
 
 /**
  * Codec to encode OmKeyInfo as byte array.
  */
 public class OmKeyInfoCodec implements Codec<OmKeyInfo> {
+  private static final Logger LOG =
+      LoggerFactory.getLogger(OmKeyInfoCodec.class);
+
+  private final boolean ignorePipeline;
+  public OmKeyInfoCodec(boolean ignorePipeline) {
+    this.ignorePipeline = ignorePipeline;
+    LOG.info("OmKeyInfoCodec ignorePipeline = " + ignorePipeline);
+  }
 
   @Override
   public byte[] toPersistedFormat(OmKeyInfo object) throws IOException {
     Preconditions
         .checkNotNull(object, "Null object can't be converted to byte array.");
-    return object.getProtobuf().toByteArray();
+    return object.getProtobuf(ignorePipeline).toByteArray();
   }
 
   @Override
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/RepeatedOmKeyInfoCodec.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/RepeatedOmKeyInfoCodec.java
index 1907b79..9156fdf 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/RepeatedOmKeyInfoCodec.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/codec/RepeatedOmKeyInfoCodec.java
@@ -22,6 +22,8 @@ import org.apache.hadoop.hdds.utils.db.Codec;
 import org.apache.hadoop.ozone.om.helpers.RepeatedOmKeyInfo;
 import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos
     .RepeatedKeyInfo;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
 
 import java.io.IOException;
 
@@ -29,12 +31,21 @@ import java.io.IOException;
  * Codec to encode RepeatedOmKeyInfo as byte array.
  */
 public class RepeatedOmKeyInfoCodec implements Codec<RepeatedOmKeyInfo> {
+  private static final Logger LOG =
+      LoggerFactory.getLogger(RepeatedOmKeyInfoCodec.class);
+
+  private final boolean ignorePipeline;
+  public RepeatedOmKeyInfoCodec(boolean ignorePipeline) {
+    this.ignorePipeline = ignorePipeline;
+    LOG.info("RepeatedOmKeyInfoCodec ignorePipeline = " + ignorePipeline);
+  }
+
   @Override
   public byte[] toPersistedFormat(RepeatedOmKeyInfo object)
       throws IOException {
     Preconditions.checkNotNull(object,
         "Null object can't be converted to byte array.");
-    return object.getProto().toByteArray();
+    return object.getProto(ignorePipeline).toByteArray();
   }
 
   @Override
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/request/file/OMFileCreateRequest.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/request/file/OMFileCreateRequest.java
index 7327626..3226f78 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/request/file/OMFileCreateRequest.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/request/file/OMFileCreateRequest.java
@@ -225,6 +225,10 @@ public class OMFileCreateRequest extends OMKeyRequest {
       OmKeyInfo dbKeyInfo = omMetadataManager.getKeyTable()
           .getIfExist(ozoneKey);
 
+      if (dbKeyInfo != null) {
+        ozoneManager.getKeyManager().refresh(dbKeyInfo);
+      }
+
       OMFileRequest.OMPathInfo pathInfo =
           OMFileRequest.verifyFilesInPath(omMetadataManager, volumeName,
               bucketName, keyName, Paths.get(keyName));
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/request/key/OMKeyCreateRequest.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/request/key/OMKeyCreateRequest.java
index 8927c19..1b712fb 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/request/key/OMKeyCreateRequest.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/request/key/OMKeyCreateRequest.java
@@ -233,6 +233,10 @@ public class OMKeyCreateRequest extends OMKeyRequest {
       OmKeyInfo dbKeyInfo =
           omMetadataManager.getKeyTable().getIfExist(dbKeyName);
 
+      if (dbKeyInfo != null) {
+        ozoneManager.getKeyManager().refresh(dbKeyInfo);
+      }
+
       OmBucketInfo bucketInfo = omMetadataManager.getBucketTable().get(
           omMetadataManager.getBucketKey(volumeName, bucketName));
 
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/protocolPB/OzoneManagerRequestHandler.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/protocolPB/OzoneManagerRequestHandler.java
index 3183aa1..35ab275 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/protocolPB/OzoneManagerRequestHandler.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/protocolPB/OzoneManagerRequestHandler.java
@@ -417,7 +417,7 @@ public class OzoneManagerRequestHandler implements RequestHandler {
         request.getMaxKeys());
 
     for (RepeatedOmKeyInfo key: deletedKeys) {
-      resp.addDeletedKeys(key.getProto());
+      resp.addDeletedKeys(key.getProto(false));
     }
 
     return resp.build();
diff --git a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/TestKeyManagerUnit.java b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/TestKeyManagerUnit.java
index 7f31b24..d06e43d 100644
--- a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/TestKeyManagerUnit.java
+++ b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/TestKeyManagerUnit.java
@@ -417,26 +417,6 @@ public class TestKeyManagerUnit {
         .setBucketName("bucketOne")
         .setKeyName("keyOne");
 
-    keyArgs.setRefreshPipeline(false);
-    final OmKeyInfo oldKeyInfo = manager
-        .lookupFile(keyArgs.build(), "test");
-
-    final OmKeyLocationInfo oldBlockLocation = oldKeyInfo
-        .getLatestVersionLocations().getBlocksLatestVersionOnly().get(0);
-
-    Assert.assertEquals(1L, oldBlockLocation.getContainerID());
-    Assert.assertEquals(1L, oldBlockLocation
-        .getBlockID().getLocalID());
-    Assert.assertEquals(pipelineOne.getId(),
-        oldBlockLocation.getPipeline().getId());
-    Assert.assertTrue(oldBlockLocation.getPipeline()
-        .getNodes().contains(dnOne));
-    Assert.assertTrue(oldBlockLocation.getPipeline()
-        .getNodes().contains(dnTwo));
-    Assert.assertTrue(oldBlockLocation.getPipeline()
-        .getNodes().contains(dnThree));
-
-    keyArgs.setRefreshPipeline(true);
     final OmKeyInfo newKeyInfo = manager
         .lookupFile(keyArgs.build(), "test");
 
diff --git a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/codec/TestOmKeyInfoCodec.java b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/codec/TestOmKeyInfoCodec.java
new file mode 100644
index 0000000..39c35f6
--- /dev/null
+++ b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/codec/TestOmKeyInfoCodec.java
@@ -0,0 +1,116 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.ozone.om.codec;
+
+import org.apache.hadoop.hdds.client.BlockID;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos;
+import org.apache.hadoop.hdds.scm.TestUtils;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.ozone.om.helpers.OmKeyInfo;
+import org.apache.hadoop.ozone.om.helpers.OmKeyLocationInfo;
+import org.apache.hadoop.ozone.om.helpers.OmKeyLocationInfoGroup;
+import org.apache.hadoop.util.Time;
+import org.junit.Test;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.List;
+
+import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertNull;
+import static org.junit.Assert.fail;
+
+/**
+ * This class tests OmKeyInfoCodec.
+ */
+public class TestOmKeyInfoCodec {
+  private final String volume = "hadoop";
+  private final String bucket = "ozone";
+  private final String keyName = "user/root/terasort/10G-input-6/part-m-00037";
+
+
+  private OmKeyInfo getKeyInfo(int chunkNum) {
+    List<OmKeyLocationInfo> omKeyLocationInfoList = new ArrayList<>();
+    Pipeline pipeline = TestUtils.getRandomPipeline();
+    for (int i = 0; i < chunkNum; i++) {
+      BlockID blockID = new BlockID(i, i);
+      OmKeyLocationInfo keyLocationInfo = new OmKeyLocationInfo.Builder()
+          .setBlockID(blockID)
+          .setPipeline(pipeline)
+          .build();
+      omKeyLocationInfoList.add(keyLocationInfo);
+    }
+    OmKeyLocationInfoGroup omKeyLocationInfoGroup = new
+        OmKeyLocationInfoGroup(0, omKeyLocationInfoList);
+    return new OmKeyInfo.Builder()
+        .setCreationTime(Time.now())
+        .setModificationTime(Time.now())
+        .setReplicationType(HddsProtos.ReplicationType.RATIS)
+        .setReplicationFactor(HddsProtos.ReplicationFactor.THREE)
+        .setVolumeName(volume)
+        .setBucketName(bucket)
+        .setKeyName(keyName)
+        .setObjectID(Time.now())
+        .setUpdateID(Time.now())
+        .setDataSize(100)
+        .setOmKeyLocationInfos(
+            Collections.singletonList(omKeyLocationInfoGroup))
+        .build();
+  }
+
+  @Test
+  public void test() {
+    testOmKeyInfoCodecWithoutPipeline(1);
+    testOmKeyInfoCodecWithoutPipeline(2);
+    testOmKeyInfoCodecCompatibility(1);
+    testOmKeyInfoCodecCompatibility(2);
+  }
+
+  public void testOmKeyInfoCodecWithoutPipeline(int chunkNum) {
+    OmKeyInfoCodec codec = new OmKeyInfoCodec(true);
+    OmKeyInfo originKey = getKeyInfo(chunkNum);
+    try {
+      byte[] rawData = codec.toPersistedFormat(originKey);
+      OmKeyInfo key = codec.fromPersistedFormat(rawData);
+      System.out.println("Chunk number = " + chunkNum +
+          ", Serialized key size without pipeline = " + rawData.length);
+      assertNull(key.getLatestVersionLocations().getLocationList().get(0)
+          .getPipeline());
+    } catch (IOException e) {
+      fail("Should success");
+    }
+  }
+
+  public void testOmKeyInfoCodecCompatibility(int chunkNum) {
+    OmKeyInfoCodec codecWithoutPipeline = new OmKeyInfoCodec(true);
+    OmKeyInfoCodec codecWithPipeline = new OmKeyInfoCodec(false);
+    OmKeyInfo originKey = getKeyInfo(chunkNum);
+    try {
+      byte[] rawData = codecWithPipeline.toPersistedFormat(originKey);
+      OmKeyInfo key = codecWithoutPipeline.fromPersistedFormat(rawData);
+      System.out.println("Chunk number = " + chunkNum +
+          ", Serialized key size with pipeline = " + rawData.length);
+      assertNotNull(key.getLatestVersionLocations().getLocationList().get(0)
+          .getPipeline());
+    } catch (IOException e) {
+      fail("Should success");
+    }
+  }
+}
diff --git a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/codec/TestRepeatedOmKeyInfoCodec.java b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/codec/TestRepeatedOmKeyInfoCodec.java
new file mode 100644
index 0000000..0eb87b8
--- /dev/null
+++ b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/codec/TestRepeatedOmKeyInfoCodec.java
@@ -0,0 +1,121 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.ozone.om.codec;
+
+import org.apache.hadoop.hdds.client.BlockID;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos;
+import org.apache.hadoop.hdds.scm.TestUtils;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.ozone.om.helpers.OmKeyInfo;
+import org.apache.hadoop.ozone.om.helpers.OmKeyLocationInfo;
+import org.apache.hadoop.ozone.om.helpers.OmKeyLocationInfoGroup;
+import org.apache.hadoop.ozone.om.helpers.RepeatedOmKeyInfo;
+import org.apache.hadoop.util.Time;
+import org.junit.Test;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.List;
+
+import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertNull;
+import static org.junit.Assert.fail;
+
+/**
+ * This class tests RepeatedOmKeyInfoCodec.
+ */
+public class TestRepeatedOmKeyInfoCodec {
+  private final String volume = "hadoop";
+  private final String bucket = "ozone";
+  private final String keyName = "user/root/terasort/10G-input-6/part-m-00037";
+
+
+  private OmKeyInfo getKeyInfo(int chunkNum) {
+    List<OmKeyLocationInfo> omKeyLocationInfoList = new ArrayList<>();
+    Pipeline pipeline = TestUtils.getRandomPipeline();
+    for (int i = 0; i < chunkNum; i++) {
+      BlockID blockID = new BlockID(i, i);
+      OmKeyLocationInfo keyLocationInfo = new OmKeyLocationInfo.Builder()
+          .setBlockID(blockID)
+          .setPipeline(pipeline)
+          .build();
+      omKeyLocationInfoList.add(keyLocationInfo);
+    }
+    OmKeyLocationInfoGroup omKeyLocationInfoGroup = new
+        OmKeyLocationInfoGroup(0, omKeyLocationInfoList);
+    return new OmKeyInfo.Builder()
+        .setCreationTime(Time.now())
+        .setModificationTime(Time.now())
+        .setReplicationType(HddsProtos.ReplicationType.RATIS)
+        .setReplicationFactor(HddsProtos.ReplicationFactor.THREE)
+        .setVolumeName(volume)
+        .setBucketName(bucket)
+        .setKeyName(keyName)
+        .setObjectID(Time.now())
+        .setUpdateID(Time.now())
+        .setDataSize(100)
+        .setOmKeyLocationInfos(
+            Collections.singletonList(omKeyLocationInfoGroup))
+        .build();
+  }
+
+  @Test
+  public void test() {
+    testWithoutPipeline(1);
+    testWithoutPipeline(2);
+    testCompatibility(1);
+    testCompatibility(2);
+  }
+
+  public void testWithoutPipeline(int chunkNum) {
+    RepeatedOmKeyInfoCodec codec = new RepeatedOmKeyInfoCodec(true);
+    OmKeyInfo originKey = getKeyInfo(chunkNum);
+    RepeatedOmKeyInfo repeatedOmKeyInfo = new RepeatedOmKeyInfo(originKey);
+    try {
+      byte[] rawData = codec.toPersistedFormat(repeatedOmKeyInfo);
+      RepeatedOmKeyInfo key = codec.fromPersistedFormat(rawData);
+      System.out.println("Chunk number = " + chunkNum +
+          ", Serialized key size without pipeline = " + rawData.length);
+      assertNull(key.getOmKeyInfoList().get(0).getLatestVersionLocations()
+          .getLocationList().get(0).getPipeline());
+    } catch (IOException e) {
+      fail("Should success");
+    }
+  }
+
+  public void testCompatibility(int chunkNum) {
+    RepeatedOmKeyInfoCodec codecWithoutPipeline =
+        new RepeatedOmKeyInfoCodec(true);
+    RepeatedOmKeyInfoCodec codecWithPipeline =
+        new RepeatedOmKeyInfoCodec(false);
+    OmKeyInfo originKey = getKeyInfo(chunkNum);
+    RepeatedOmKeyInfo repeatedOmKeyInfo = new RepeatedOmKeyInfo(originKey);
+    try {
+      byte[] rawData = codecWithPipeline.toPersistedFormat(repeatedOmKeyInfo);
+      RepeatedOmKeyInfo key = codecWithoutPipeline.fromPersistedFormat(rawData);
+      System.out.println("Chunk number = " + chunkNum +
+          ", Serialized key size with pipeline = " + rawData.length);
+      assertNotNull(key.getOmKeyInfoList().get(0).getLatestVersionLocations()
+          .getLocationList().get(0).getPipeline());
+    } catch (IOException e) {
+      fail("Should success");
+    }
+  }
+}
diff --git a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/request/key/TestOMKeyRequest.java b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/request/key/TestOMKeyRequest.java
index dd6caf4..cb35e2b 100644
--- a/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/request/key/TestOMKeyRequest.java
+++ b/hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/om/request/key/TestOMKeyRequest.java
@@ -24,6 +24,8 @@ import java.util.UUID;
 
 import org.apache.commons.lang3.tuple.Pair;
 import org.apache.hadoop.ozone.om.ResolvedBucket;
+import org.apache.hadoop.ozone.om.KeyManager;
+import org.apache.hadoop.ozone.om.KeyManagerImpl;
 import org.apache.hadoop.ozone.om.ratis.utils.OzoneManagerDoubleBufferHelper;
 import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.KeyArgs;
 import org.junit.After;
@@ -66,6 +68,7 @@ public class TestOMKeyRequest {
   public TemporaryFolder folder = new TemporaryFolder();
 
   protected OzoneManager ozoneManager;
+  protected KeyManager keyManager;
   protected OMMetrics omMetrics;
   protected OMMetadataManager omMetadataManager;
   protected AuditLogger auditLogger;
@@ -113,6 +116,8 @@ public class TestOMKeyRequest {
     ozoneBlockTokenSecretManager =
         Mockito.mock(OzoneBlockTokenSecretManager.class);
     scmBlockLocationProtocol = Mockito.mock(ScmBlockLocationProtocol.class);
+    keyManager = new KeyManagerImpl(ozoneManager, scmClient, ozoneConfiguration,
+        "");
     when(ozoneManager.getScmClient()).thenReturn(scmClient);
     when(ozoneManager.getBlockTokenSecretManager())
         .thenReturn(ozoneBlockTokenSecretManager);
@@ -121,6 +126,7 @@ public class TestOMKeyRequest {
     when(ozoneManager.isGrpcBlockTokenEnabled()).thenReturn(false);
     when(ozoneManager.getOMNodeId()).thenReturn(UUID.randomUUID().toString());
     when(scmClient.getBlockClient()).thenReturn(scmBlockLocationProtocol);
+    when(ozoneManager.getKeyManager()).thenReturn(keyManager);
 
     Pipeline pipeline = Pipeline.newBuilder()
         .setState(Pipeline.PipelineState.OPEN)
diff --git a/hadoop-ozone/tools/src/main/java/org/apache/hadoop/ozone/fsck/ContainerMapper.java b/hadoop-ozone/tools/src/main/java/org/apache/hadoop/ozone/fsck/ContainerMapper.java
index 81ff0ea..fe73e35 100644
--- a/hadoop-ozone/tools/src/main/java/org/apache/hadoop/ozone/fsck/ContainerMapper.java
+++ b/hadoop-ozone/tools/src/main/java/org/apache/hadoop/ozone/fsck/ContainerMapper.java
@@ -93,7 +93,7 @@ public class ContainerMapper {
             Table.KeyValue<String, OmKeyInfo> keyValue =
                 keyValueTableIterator.next();
             OmKeyInfo omKeyInfo = keyValue.getValue();
-            byte[] value = omKeyInfo.getProtobuf().toByteArray();
+            byte[] value = omKeyInfo.getProtobuf(true).toByteArray();
             OmKeyInfo keyInfo = OmKeyInfo.getFromProtobuf(
                 OzoneManagerProtocolProtos.KeyInfo.parseFrom(value));
             for (OmKeyLocationInfoGroup keyLocationInfoGroup : keyInfo


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