|
@@ -73,12 +73,6 @@ import org.apache.hadoop.ozone.om.helpers.OmMultipartUploadListParts;
|
|
|
import org.apache.hadoop.ozone.om.helpers.OmPartInfo;
|
|
|
import org.apache.hadoop.ozone.om.helpers.OpenKeySession;
|
|
|
import org.apache.hadoop.ozone.om.helpers.OzoneFSUtils;
|
|
|
-import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos
|
|
|
- .KeyArgs;
|
|
|
-import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos
|
|
|
- .KeyInfo;
|
|
|
-import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos
|
|
|
- .KeyLocation;
|
|
|
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.OzoneAclInfo;
|
|
|
import org.apache.hadoop.ozone.security.OzoneBlockTokenSecretManager;
|
|
|
import org.apache.hadoop.ozone.security.acl.OzoneObj;
|
|
@@ -156,7 +150,6 @@ public class KeyManagerImpl implements KeyManager {
|
|
|
|
|
|
private final KeyProviderCryptoExtension kmsProvider;
|
|
|
|
|
|
- private final boolean isRatisEnabled;
|
|
|
|
|
|
public KeyManagerImpl(ScmBlockLocationProtocol scmBlockClient,
|
|
|
OMMetadataManager metadataManager, OzoneConfiguration conf, String omId,
|
|
@@ -186,9 +179,6 @@ public class KeyManagerImpl implements KeyManager {
|
|
|
HDDS_BLOCK_TOKEN_ENABLED,
|
|
|
HDDS_BLOCK_TOKEN_ENABLED_DEFAULT);
|
|
|
this.kmsProvider = kmsProvider;
|
|
|
- this.isRatisEnabled = conf.getBoolean(
|
|
|
- OMConfigKeys.OZONE_OM_RATIS_ENABLE_KEY,
|
|
|
- OMConfigKeys.OZONE_OM_RATIS_ENABLE_DEFAULT);
|
|
|
}
|
|
|
|
|
|
@Override
|
|
@@ -266,36 +256,6 @@ public class KeyManagerImpl implements KeyManager {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- @Override
|
|
|
- public OmKeyLocationInfo addAllocatedBlock(OmKeyArgs args, long clientID,
|
|
|
- KeyLocation keyLocation) throws IOException {
|
|
|
- Preconditions.checkNotNull(args);
|
|
|
- Preconditions.checkNotNull(keyLocation);
|
|
|
-
|
|
|
-
|
|
|
- String volumeName = args.getVolumeName();
|
|
|
- String bucketName = args.getBucketName();
|
|
|
- String keyName = args.getKeyName();
|
|
|
- validateBucket(volumeName, bucketName);
|
|
|
- String openKey = metadataManager.getOpenKey(
|
|
|
- volumeName, bucketName, keyName, clientID);
|
|
|
-
|
|
|
- OmKeyInfo keyInfo = metadataManager.getOpenKeyTable().get(openKey);
|
|
|
- if (keyInfo == null) {
|
|
|
- LOG.error("Allocate block for a key not in open status in meta store" +
|
|
|
- " /{}/{}/{} with ID {}", volumeName, bucketName, keyName, clientID);
|
|
|
- throw new OMException("Open Key not found",
|
|
|
- KEY_NOT_FOUND);
|
|
|
- }
|
|
|
-
|
|
|
- OmKeyLocationInfo omKeyLocationInfo =
|
|
|
- OmKeyLocationInfo.getFromProtobuf(keyLocation);
|
|
|
- keyInfo.appendNewBlocks(Collections.singletonList(omKeyLocationInfo), true);
|
|
|
- keyInfo.updateModifcationTime();
|
|
|
- metadataManager.getOpenKeyTable().put(openKey, keyInfo);
|
|
|
- return omKeyLocationInfo;
|
|
|
- }
|
|
|
-
|
|
|
@Override
|
|
|
public OmKeyLocationInfo allocateBlock(OmKeyArgs args, long clientID,
|
|
|
ExcludeList excludeList) throws IOException {
|
|
@@ -322,13 +282,10 @@ public class KeyManagerImpl implements KeyManager {
|
|
|
List<OmKeyLocationInfo> locationInfos =
|
|
|
allocateBlock(keyInfo, excludeList, scmBlockSize);
|
|
|
|
|
|
- // If om is not managing via ratis, write to db, otherwise write to DB
|
|
|
- // will happen via ratis apply transaction.
|
|
|
- if (!isRatisEnabled) {
|
|
|
- keyInfo.appendNewBlocks(locationInfos, true);
|
|
|
- keyInfo.updateModifcationTime();
|
|
|
- metadataManager.getOpenKeyTable().put(openKey, keyInfo);
|
|
|
- }
|
|
|
+ keyInfo.appendNewBlocks(locationInfos, true);
|
|
|
+ keyInfo.updateModifcationTime();
|
|
|
+ metadataManager.getOpenKeyTable().put(openKey, keyInfo);
|
|
|
+
|
|
|
return locationInfos.get(0);
|
|
|
|
|
|
}
|
|
@@ -497,11 +454,8 @@ public class KeyManagerImpl implements KeyManager {
|
|
|
keyInfo.appendNewBlocks(locationInfos, true);
|
|
|
}
|
|
|
|
|
|
- // When OM is not managed via ratis we should write in to Om db in
|
|
|
- // openKey call.
|
|
|
- if (!isRatisEnabled) {
|
|
|
- metadataManager.getOpenKeyTable().put(openKey, keyInfo);
|
|
|
- }
|
|
|
+ metadataManager.getOpenKeyTable().put(openKey, keyInfo);
|
|
|
+
|
|
|
}
|
|
|
|
|
|
private OmKeyInfo prepareKeyInfo(
|
|
@@ -555,41 +509,6 @@ public class KeyManagerImpl implements KeyManager {
|
|
|
return createKeyInfo(args, locations, factor, type, size, encInfo);
|
|
|
}
|
|
|
|
|
|
- public void applyOpenKey(KeyArgs omKeyArgs,
|
|
|
- KeyInfo keyInfo, long clientID) throws IOException {
|
|
|
- Preconditions.checkNotNull(omKeyArgs);
|
|
|
- String volumeName = omKeyArgs.getVolumeName();
|
|
|
- String bucketName = omKeyArgs.getBucketName();
|
|
|
-
|
|
|
- // Do we need to call again validateBucket, as this is just called after
|
|
|
- // start Transaction from applyTransaction. Can we remove this double
|
|
|
- // check?
|
|
|
- validateBucket(volumeName, bucketName);
|
|
|
-
|
|
|
- metadataManager.getLock().acquireLock(BUCKET_LOCK, volumeName, bucketName);
|
|
|
- String keyName = omKeyArgs.getKeyName();
|
|
|
-
|
|
|
- // TODO: here if on OM machines clocks are skewed and there is a chance
|
|
|
- // for override of the openKey entries.
|
|
|
- try {
|
|
|
- String openKey = metadataManager.getOpenKey(
|
|
|
- volumeName, bucketName, keyName, clientID);
|
|
|
-
|
|
|
- OmKeyInfo omKeyInfo = OmKeyInfo.getFromProtobuf(keyInfo);
|
|
|
-
|
|
|
- metadataManager.getOpenKeyTable().put(openKey,
|
|
|
- omKeyInfo);
|
|
|
- } catch (IOException ex) {
|
|
|
- LOG.error("Apply Open Key failed for volume:{} bucket:{} key:{}",
|
|
|
- volumeName, bucketName, keyName, ex);
|
|
|
- throw new OMException(ex.getMessage(),
|
|
|
- ResultCodes.KEY_ALLOCATION_ERROR);
|
|
|
- } finally {
|
|
|
- metadataManager.getLock().releaseLock(BUCKET_LOCK, volumeName,
|
|
|
- bucketName);
|
|
|
- }
|
|
|
- }
|
|
|
-
|
|
|
/**
|
|
|
* Create OmKeyInfo object.
|
|
|
* @param keyArgs
|