HDDS-288. Fix bugs in OpenContainerBlockMap. Contributed by Tsz Wo Nicholas Sze.
This commit is contained in:
parent
3d3158cea4
commit
3c4fbc635e
@ -21,22 +21,52 @@
|
|||||||
import com.google.common.annotations.VisibleForTesting;
|
import com.google.common.annotations.VisibleForTesting;
|
||||||
import com.google.common.base.Preconditions;
|
import com.google.common.base.Preconditions;
|
||||||
import org.apache.hadoop.hdds.client.BlockID;
|
import org.apache.hadoop.hdds.client.BlockID;
|
||||||
import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos;
|
import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.ChunkInfo;
|
||||||
import org.apache.hadoop.ozone.container.common.helpers.KeyData;
|
import org.apache.hadoop.ozone.container.common.helpers.KeyData;
|
||||||
|
|
||||||
import java.io.IOException;
|
import java.util.ArrayList;
|
||||||
import java.util.HashMap;
|
import java.util.Collections;
|
||||||
import java.util.LinkedHashMap;
|
|
||||||
import java.util.List;
|
import java.util.List;
|
||||||
|
import java.util.Optional;
|
||||||
import java.util.concurrent.ConcurrentHashMap;
|
import java.util.concurrent.ConcurrentHashMap;
|
||||||
import java.util.stream.Collectors;
|
import java.util.concurrent.ConcurrentMap;
|
||||||
|
import java.util.function.Function;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
* Map: containerId -> (localId -> KeyData).
|
||||||
|
* The outer container map does not entail locking for a better performance.
|
||||||
|
* The inner {@link KeyDataMap} is synchronized.
|
||||||
|
*
|
||||||
* This class will maintain list of open keys per container when closeContainer
|
* This class will maintain list of open keys per container when closeContainer
|
||||||
* command comes, it should autocommit all open keys of a open container before
|
* command comes, it should autocommit all open keys of a open container before
|
||||||
* marking the container as closed.
|
* marking the container as closed.
|
||||||
*/
|
*/
|
||||||
public class OpenContainerBlockMap {
|
public class OpenContainerBlockMap {
|
||||||
|
/**
|
||||||
|
* Map: localId -> KeyData.
|
||||||
|
*
|
||||||
|
* In order to support {@link #getAll()}, the update operations are synchronized.
|
||||||
|
*/
|
||||||
|
static class KeyDataMap {
|
||||||
|
private final ConcurrentMap<Long, KeyData> blocks = new ConcurrentHashMap<>();
|
||||||
|
|
||||||
|
KeyData get(long localId) {
|
||||||
|
return blocks.get(localId);
|
||||||
|
}
|
||||||
|
|
||||||
|
synchronized int removeAndGetSize(long localId) {
|
||||||
|
blocks.remove(localId);
|
||||||
|
return blocks.size();
|
||||||
|
}
|
||||||
|
|
||||||
|
synchronized KeyData computeIfAbsent(long localId, Function<Long, KeyData> f) {
|
||||||
|
return blocks.computeIfAbsent(localId, f);
|
||||||
|
}
|
||||||
|
|
||||||
|
synchronized List<KeyData> getAll() {
|
||||||
|
return new ArrayList<>(blocks.values());
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* TODO : We may construct the openBlockMap by reading the Block Layout
|
* TODO : We may construct the openBlockMap by reading the Block Layout
|
||||||
@ -46,15 +76,8 @@ public class OpenContainerBlockMap {
|
|||||||
*
|
*
|
||||||
* For now, we will track all open blocks of a container in the blockMap.
|
* For now, we will track all open blocks of a container in the blockMap.
|
||||||
*/
|
*/
|
||||||
private final ConcurrentHashMap<Long, HashMap<Long, KeyData>>
|
private final ConcurrentMap<Long, KeyDataMap> containers = new ConcurrentHashMap<>();
|
||||||
openContainerBlockMap;
|
|
||||||
|
|
||||||
/**
|
|
||||||
* Constructs OpenContainerBlockMap.
|
|
||||||
*/
|
|
||||||
public OpenContainerBlockMap() {
|
|
||||||
openContainerBlockMap = new ConcurrentHashMap<>();
|
|
||||||
}
|
|
||||||
/**
|
/**
|
||||||
* Removes the Container matching with specified containerId.
|
* Removes the Container matching with specified containerId.
|
||||||
* @param containerId containerId
|
* @param containerId containerId
|
||||||
@ -62,73 +85,27 @@ public OpenContainerBlockMap() {
|
|||||||
public void removeContainer(long containerId) {
|
public void removeContainer(long containerId) {
|
||||||
Preconditions
|
Preconditions
|
||||||
.checkState(containerId >= 0, "Container Id cannot be negative.");
|
.checkState(containerId >= 0, "Container Id cannot be negative.");
|
||||||
openContainerBlockMap.computeIfPresent(containerId, (k, v) -> null);
|
containers.remove(containerId);
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
public void addChunk(BlockID blockID, ChunkInfo info) {
|
||||||
* updates the chunkInfoList in case chunk is added or deleted
|
|
||||||
* @param blockID id of the block.
|
|
||||||
* @param info - Chunk Info
|
|
||||||
* @param remove if true, deletes the chunkInfo list otherwise appends to the
|
|
||||||
* chunkInfo List
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
public synchronized void updateOpenKeyMap(BlockID blockID,
|
|
||||||
ContainerProtos.ChunkInfo info, boolean remove) throws IOException {
|
|
||||||
if (remove) {
|
|
||||||
deleteChunkFromMap(blockID, info);
|
|
||||||
} else {
|
|
||||||
addChunkToMap(blockID, info);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
private KeyData getKeyData(ContainerProtos.ChunkInfo info, BlockID blockID)
|
|
||||||
throws IOException {
|
|
||||||
KeyData keyData = new KeyData(blockID);
|
|
||||||
keyData.addMetadata("TYPE", "KEY");
|
|
||||||
keyData.addChunk(info);
|
|
||||||
return keyData;
|
|
||||||
}
|
|
||||||
|
|
||||||
private void addChunkToMap(BlockID blockID, ContainerProtos.ChunkInfo info)
|
|
||||||
throws IOException {
|
|
||||||
Preconditions.checkNotNull(info);
|
Preconditions.checkNotNull(info);
|
||||||
long containerId = blockID.getContainerID();
|
containers.computeIfAbsent(blockID.getContainerID(), id -> new KeyDataMap())
|
||||||
long localID = blockID.getLocalID();
|
.computeIfAbsent(blockID.getLocalID(), id -> new KeyData(blockID))
|
||||||
|
.addChunk(info);
|
||||||
KeyData keyData = openContainerBlockMap.computeIfAbsent(containerId,
|
|
||||||
emptyMap -> new LinkedHashMap<Long, KeyData>())
|
|
||||||
.putIfAbsent(localID, getKeyData(info, blockID));
|
|
||||||
// KeyData != null means the block already exist
|
|
||||||
if (keyData != null) {
|
|
||||||
HashMap<Long, KeyData> keyDataSet =
|
|
||||||
openContainerBlockMap.get(containerId);
|
|
||||||
keyDataSet.putIfAbsent(blockID.getLocalID(), getKeyData(info, blockID));
|
|
||||||
keyDataSet.computeIfPresent(blockID.getLocalID(), (key, value) -> {
|
|
||||||
value.addChunk(info);
|
|
||||||
return value;
|
|
||||||
});
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* removes the chunks from the chunkInfo list for the given block.
|
* Removes the chunk from the chunkInfo list for the given block.
|
||||||
* @param blockID id of the block
|
* @param blockID id of the block
|
||||||
* @param chunkInfo chunk info.
|
* @param chunkInfo chunk info.
|
||||||
*/
|
*/
|
||||||
private synchronized void deleteChunkFromMap(BlockID blockID,
|
public void removeChunk(BlockID blockID, ChunkInfo chunkInfo) {
|
||||||
ContainerProtos.ChunkInfo chunkInfo) {
|
|
||||||
Preconditions.checkNotNull(chunkInfo);
|
Preconditions.checkNotNull(chunkInfo);
|
||||||
Preconditions.checkNotNull(blockID);
|
Preconditions.checkNotNull(blockID);
|
||||||
HashMap<Long, KeyData> keyDataMap =
|
Optional.ofNullable(containers.get(blockID.getContainerID()))
|
||||||
openContainerBlockMap.get(blockID.getContainerID());
|
.map(blocks -> blocks.get(blockID.getLocalID()))
|
||||||
if (keyDataMap != null) {
|
.ifPresent(keyData -> keyData.removeChunk(chunkInfo));
|
||||||
long localId = blockID.getLocalID();
|
|
||||||
KeyData keyData = keyDataMap.get(localId);
|
|
||||||
if (keyData != null) {
|
|
||||||
keyData.removeChunk(chunkInfo);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
@ -137,31 +114,23 @@ private synchronized void deleteChunkFromMap(BlockID blockID,
|
|||||||
* @return List of open Keys(blocks)
|
* @return List of open Keys(blocks)
|
||||||
*/
|
*/
|
||||||
public List<KeyData> getOpenKeys(long containerId) {
|
public List<KeyData> getOpenKeys(long containerId) {
|
||||||
HashMap<Long, KeyData> keyDataHashMap =
|
return Optional.ofNullable(containers.get(containerId))
|
||||||
openContainerBlockMap.get(containerId);
|
.map(KeyDataMap::getAll)
|
||||||
return keyDataHashMap == null ? null :
|
.orElseGet(Collections::emptyList);
|
||||||
keyDataHashMap.values().stream().collect(Collectors.toList());
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* removes the block from the block map.
|
* removes the block from the block map.
|
||||||
* @param blockID
|
* @param blockID
|
||||||
*/
|
*/
|
||||||
public synchronized void removeFromKeyMap(BlockID blockID) {
|
public void removeFromKeyMap(BlockID blockID) {
|
||||||
Preconditions.checkNotNull(blockID);
|
Preconditions.checkNotNull(blockID);
|
||||||
HashMap<Long, KeyData> keyDataMap =
|
containers.computeIfPresent(blockID.getContainerID(), (containerId, blocks)
|
||||||
openContainerBlockMap.get(blockID.getContainerID());
|
-> blocks.removeAndGetSize(blockID.getLocalID()) == 0? null: blocks);
|
||||||
if (keyDataMap != null) {
|
|
||||||
keyDataMap.remove(blockID.getLocalID());
|
|
||||||
if (keyDataMap.size() == 0) {
|
|
||||||
removeContainer(blockID.getContainerID());
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@VisibleForTesting
|
@VisibleForTesting
|
||||||
public ConcurrentHashMap<Long,
|
KeyDataMap getKeyDataMap(long containerId) {
|
||||||
HashMap<Long, KeyData>> getContainerOpenKeyMap() {
|
return containers.get(containerId);
|
||||||
return openContainerBlockMap;
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
@ -435,10 +435,8 @@ private void commitPendingKeys(KeyValueContainer kvContainer)
|
|||||||
long containerId = kvContainer.getContainerData().getContainerID();
|
long containerId = kvContainer.getContainerData().getContainerID();
|
||||||
List<KeyData> pendingKeys =
|
List<KeyData> pendingKeys =
|
||||||
this.openContainerBlockMap.getOpenKeys(containerId);
|
this.openContainerBlockMap.getOpenKeys(containerId);
|
||||||
if (pendingKeys != null) {
|
for(KeyData keyData : pendingKeys) {
|
||||||
for (KeyData keyData : pendingKeys) {
|
commitKey(keyData, kvContainer);
|
||||||
commitKey(keyData, kvContainer);
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
@ -598,7 +596,7 @@ ContainerCommandResponseProto handleDeleteChunk(
|
|||||||
Preconditions.checkNotNull(chunkInfo);
|
Preconditions.checkNotNull(chunkInfo);
|
||||||
|
|
||||||
chunkManager.deleteChunk(kvContainer, blockID, chunkInfo);
|
chunkManager.deleteChunk(kvContainer, blockID, chunkInfo);
|
||||||
openContainerBlockMap.updateOpenKeyMap(blockID, chunkInfoProto, true);
|
openContainerBlockMap.removeChunk(blockID, chunkInfoProto);
|
||||||
} catch (StorageContainerException ex) {
|
} catch (StorageContainerException ex) {
|
||||||
return ContainerUtils.logAndReturnError(LOG, ex, request);
|
return ContainerUtils.logAndReturnError(LOG, ex, request);
|
||||||
} catch (IOException ex) {
|
} catch (IOException ex) {
|
||||||
@ -648,7 +646,7 @@ ContainerCommandResponseProto handleWriteChunk(
|
|||||||
.getChunkData().getLen());
|
.getChunkData().getLen());
|
||||||
// the openContainerBlockMap should be updated only while writing data
|
// the openContainerBlockMap should be updated only while writing data
|
||||||
// not during COMMIT_STAGE of handling write chunk request.
|
// not during COMMIT_STAGE of handling write chunk request.
|
||||||
openContainerBlockMap.updateOpenKeyMap(blockID, chunkInfoProto, false);
|
openContainerBlockMap.addChunk(blockID, chunkInfoProto);
|
||||||
}
|
}
|
||||||
} catch (StorageContainerException ex) {
|
} catch (StorageContainerException ex) {
|
||||||
return ContainerUtils.logAndReturnError(LOG, ex, request);
|
return ContainerUtils.logAndReturnError(LOG, ex, request);
|
||||||
|
@ -162,9 +162,9 @@ public void testPutKeyWithMultipleChunks()
|
|||||||
Pipeline pipeline = createSingleNodePipeline();
|
Pipeline pipeline = createSingleNodePipeline();
|
||||||
List<ChunkInfo> chunkList = writeChunkBuilder(blockID, pipeline, 3);
|
List<ChunkInfo> chunkList = writeChunkBuilder(blockID, pipeline, 3);
|
||||||
// the key should exist in the map
|
// the key should exist in the map
|
||||||
Assert.assertTrue(
|
Assert.assertNotNull(
|
||||||
openContainerBlockMap.getContainerOpenKeyMap().get(testContainerID)
|
openContainerBlockMap.getKeyDataMap(testContainerID)
|
||||||
.containsKey(blockID.getLocalID()));
|
.get(blockID.getLocalID()));
|
||||||
KeyData keyData = new KeyData(blockID);
|
KeyData keyData = new KeyData(blockID);
|
||||||
List<ContainerProtos.ChunkInfo> chunkProtoList = new LinkedList<>();
|
List<ContainerProtos.ChunkInfo> chunkProtoList = new LinkedList<>();
|
||||||
for (ChunkInfo i : chunkList) {
|
for (ChunkInfo i : chunkList) {
|
||||||
@ -184,7 +184,7 @@ public void testPutKeyWithMultipleChunks()
|
|||||||
|
|
||||||
//the open key should be removed from Map
|
//the open key should be removed from Map
|
||||||
Assert.assertNull(
|
Assert.assertNull(
|
||||||
openContainerBlockMap.getContainerOpenKeyMap().get(testContainerID));
|
openContainerBlockMap.getKeyDataMap(testContainerID));
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
@ -196,11 +196,11 @@ public void testDeleteChunk() throws Exception {
|
|||||||
Pipeline pipeline = createSingleNodePipeline();
|
Pipeline pipeline = createSingleNodePipeline();
|
||||||
List<ChunkInfo> chunkList = writeChunkBuilder(blockID, pipeline, 3);
|
List<ChunkInfo> chunkList = writeChunkBuilder(blockID, pipeline, 3);
|
||||||
// the key should exist in the map
|
// the key should exist in the map
|
||||||
|
Assert.assertNotNull(
|
||||||
|
openContainerBlockMap.getKeyDataMap(testContainerID)
|
||||||
|
.get(blockID.getLocalID()));
|
||||||
Assert.assertTrue(
|
Assert.assertTrue(
|
||||||
openContainerBlockMap.getContainerOpenKeyMap().get(testContainerID)
|
openContainerBlockMap.getKeyDataMap(testContainerID)
|
||||||
.containsKey(blockID.getLocalID()));
|
|
||||||
Assert.assertTrue(
|
|
||||||
openContainerBlockMap.getContainerOpenKeyMap().get(testContainerID)
|
|
||||||
.get(blockID.getLocalID()).getChunks().size() == 3);
|
.get(blockID.getLocalID()).getChunks().size() == 3);
|
||||||
ContainerProtos.DeleteChunkRequestProto.Builder deleteChunkProto =
|
ContainerProtos.DeleteChunkRequestProto.Builder deleteChunkProto =
|
||||||
ContainerProtos.DeleteChunkRequestProto.newBuilder();
|
ContainerProtos.DeleteChunkRequestProto.newBuilder();
|
||||||
@ -219,7 +219,7 @@ public void testDeleteChunk() throws Exception {
|
|||||||
request.setDatanodeUuid(pipeline.getLeader().getUuidString());
|
request.setDatanodeUuid(pipeline.getLeader().getUuidString());
|
||||||
dispatcher.dispatch(request.build());
|
dispatcher.dispatch(request.build());
|
||||||
Assert.assertTrue(
|
Assert.assertTrue(
|
||||||
openContainerBlockMap.getContainerOpenKeyMap().get(testContainerID)
|
openContainerBlockMap.getKeyDataMap(testContainerID)
|
||||||
.get(blockID.getLocalID()).getChunks().size() == 2);
|
.get(blockID.getLocalID()).getChunks().size() == 2);
|
||||||
|
|
||||||
}
|
}
|
||||||
@ -234,12 +234,12 @@ public void testCloseContainer() throws Exception {
|
|||||||
List<ChunkInfo> chunkList = writeChunkBuilder(blockID, pipeline, 3);
|
List<ChunkInfo> chunkList = writeChunkBuilder(blockID, pipeline, 3);
|
||||||
|
|
||||||
Container container = containerSet.getContainer(testContainerID);
|
Container container = containerSet.getContainer(testContainerID);
|
||||||
KeyData keyData = openContainerBlockMap.getContainerOpenKeyMap().
|
KeyData keyData = openContainerBlockMap.
|
||||||
get(testContainerID).get(blockID.getLocalID());
|
getKeyDataMap(testContainerID).get(blockID.getLocalID());
|
||||||
// the key should exist in the map
|
// the key should exist in the map
|
||||||
Assert.assertTrue(
|
Assert.assertNotNull(
|
||||||
openContainerBlockMap.getContainerOpenKeyMap().get(testContainerID)
|
openContainerBlockMap.getKeyDataMap(testContainerID)
|
||||||
.containsKey(blockID.getLocalID()));
|
.get(blockID.getLocalID()));
|
||||||
Assert.assertTrue(
|
Assert.assertTrue(
|
||||||
keyData.getChunks().size() == chunkList.size());
|
keyData.getChunks().size() == chunkList.size());
|
||||||
ContainerProtos.CloseContainerRequestProto.Builder closeContainerProto =
|
ContainerProtos.CloseContainerRequestProto.Builder closeContainerProto =
|
||||||
@ -253,7 +253,7 @@ public void testCloseContainer() throws Exception {
|
|||||||
request.setDatanodeUuid(pipeline.getLeader().getUuidString());
|
request.setDatanodeUuid(pipeline.getLeader().getUuidString());
|
||||||
dispatcher.dispatch(request.build());
|
dispatcher.dispatch(request.build());
|
||||||
Assert.assertNull(
|
Assert.assertNull(
|
||||||
openContainerBlockMap.getContainerOpenKeyMap().get(testContainerID));
|
openContainerBlockMap.getKeyDataMap(testContainerID));
|
||||||
// Make sure the key got committed
|
// Make sure the key got committed
|
||||||
Assert.assertNotNull(handler.getKeyManager().getKey(container, blockID));
|
Assert.assertNotNull(handler.getKeyManager().getKey(container, blockID));
|
||||||
}
|
}
|
||||||
|
Loading…
Reference in New Issue
Block a user