HDFS-13008. Ozone: Add DN container open/close state to container report. Contributed by Xiaoyu Yao.

This commit is contained in:
Nanda kumar 2018-03-06 08:41:33 +05:30 committed by Owen O'Malley
parent bf206f014a
commit 9a914126a7
9 changed files with 222 additions and 43 deletions

View File

@ -201,10 +201,13 @@ message ContainerData {
repeated KeyValue metadata = 2;
optional string dbPath = 3;
optional string containerPath = 4;
optional bool open = 5 [default = true];
optional string hash = 6;
optional int64 bytesUsed = 7;
optional int64 size = 8;
optional int64 keyCount = 9;
//TODO: change required after we switch container ID from string to long
optional int64 containerID = 10;
optional LifeCycleState state = 11 [default = OPEN];
}
message ContainerMeta {

View File

@ -45,24 +45,26 @@ public class ContainerData {
private String dbPath; // Path to Level DB Store.
// Path to Physical file system where container and checksum are stored.
private String containerFilePath;
private boolean open;
private String hash;
private AtomicLong bytesUsed;
private long maxSize;
private Long containerID;
private OzoneProtos.LifeCycleState state;
/**
* Constructs a ContainerData Object.
*
* @param containerName - Name
*/
public ContainerData(String containerName, Configuration conf) {
public ContainerData(String containerName, Long containerID,
Configuration conf) {
this.metadata = new TreeMap<>();
this.containerName = containerName;
this.open = true;
this.maxSize = conf.getLong(ScmConfigKeys.SCM_CONTAINER_CLIENT_MAX_SIZE_KEY,
ScmConfigKeys.SCM_CONTAINER_CLIENT_MAX_SIZE_DEFAULT) * OzoneConsts.GB;
this.bytesUsed = new AtomicLong(0L);
this.containerID = containerID;
this.state = OzoneProtos.LifeCycleState.OPEN;
}
/**
@ -74,7 +76,8 @@ public ContainerData(String containerName, Configuration conf) {
public static ContainerData getFromProtBuf(
ContainerProtos.ContainerData protoData, Configuration conf)
throws IOException {
ContainerData data = new ContainerData(protoData.getName(), conf);
ContainerData data = new ContainerData(protoData.getName(),
protoData.getContainerID(), conf);
for (int x = 0; x < protoData.getMetadataCount(); x++) {
data.addMetadata(protoData.getMetadata(x).getKey(),
protoData.getMetadata(x).getValue());
@ -88,10 +91,8 @@ public static ContainerData getFromProtBuf(
data.setDBPath(protoData.getDbPath());
}
if (protoData.hasOpen()) {
data.setOpen(protoData.getOpen());
} else {
data.setOpen(true);
if (protoData.hasState()) {
data.setState(protoData.getState());
}
if(protoData.hasHash()) {
@ -117,6 +118,7 @@ public ContainerProtos.ContainerData getProtoBufMessage() {
ContainerProtos.ContainerData.Builder builder = ContainerProtos
.ContainerData.newBuilder();
builder.setName(this.getContainerName());
builder.setContainerID(this.getContainerID());
if (this.getDBPath() != null) {
builder.setDbPath(this.getDBPath());
@ -130,7 +132,7 @@ public ContainerProtos.ContainerData getProtoBufMessage() {
builder.setContainerPath(this.getContainerPath());
}
builder.setOpen(this.isOpen());
builder.setState(this.getState());
for (Map.Entry<String, String> entry : metadata.entrySet()) {
OzoneProtos.KeyValue.Builder keyValBuilder =
@ -143,6 +145,10 @@ public ContainerProtos.ContainerData getProtoBufMessage() {
builder.setBytesUsed(this.getBytesUsed());
}
if (this.getKeyCount() >= 0) {
builder.setKeyCount(this.getKeyCount());
}
if (this.getMaxSize() >= 0) {
builder.setSize(this.getMaxSize());
}
@ -245,19 +251,36 @@ public void setContainerPath(String containerPath) {
this.containerFilePath = containerPath;
}
/**
* Get container ID.
* @return - container ID.
*/
public synchronized Long getContainerID() {
return containerID;
}
public synchronized void setState(OzoneProtos.LifeCycleState state) {
this.state = state;
}
public synchronized OzoneProtos.LifeCycleState getState() {
return this.state;
}
/**
* checks if the container is open.
* @return - boolean
*/
public synchronized boolean isOpen() {
return open;
return OzoneProtos.LifeCycleState.OPEN == state;
}
/**
* Marks this container as closed.
*/
public synchronized void closeContainer() {
setOpen(false);
// TODO: closed or closing here
setState(OzoneProtos.LifeCycleState.CLOSED);
// Some thing brain dead for now. name + Time stamp of when we get the close
// container message.
@ -277,14 +300,6 @@ public void setHash(String hash) {
this.hash = hash;
}
/**
* Sets the open or closed values.
* @param open
*/
public synchronized void setOpen(boolean open) {
this.open = open;
}
public void setMaxSize(long maxSize) {
this.maxSize = maxSize;
}

View File

@ -618,7 +618,8 @@ public Pipeline getContainer(String containerName) throws IOException {
}
@VisibleForTesting
ContainerInfo getContainerInfo(String containerName) throws IOException {
public ContainerInfo getContainerInfo(String containerName)
throws IOException {
return scmContainerManager.getContainer(containerName);
}

View File

@ -25,6 +25,7 @@
import org.apache.hadoop.hdfs.ozone.protocol.proto.ContainerProtos.ContainerData;
import org.apache.hadoop.hdfs.protocol.DatanodeID;
import org.apache.hadoop.ozone.protocol.proto.OzoneProtos;
import org.apache.hadoop.ozone.scm.cli.OzoneCommandHandler;
import org.apache.hadoop.scm.client.ScmClient;
import org.apache.hadoop.scm.container.common.helpers.Pipeline;
@ -75,7 +76,9 @@ public void execute(CommandLine cmd) throws IOException {
// Print container report info.
logOut("Container Name: %s",
containerData.getName());
String openStatus = containerData.getOpen() ? "OPEN" : "CLOSED";
String openStatus =
containerData.getState() == OzoneProtos.LifeCycleState.OPEN ? "OPEN" :
"CLOSED";
logOut("Container State: %s", openStatus);
if (!containerData.getHash().isEmpty()) {
logOut("Container Hash: %s", containerData.getHash());

View File

@ -104,6 +104,7 @@ message ContainerInfo {
optional int64 readBytes = 8;
optional int64 writeBytes = 9;
required int64 containerID = 10;
optional hadoop.hdfs.ozone.LifeCycleState state = 11;
}
// The deleted blocks which are stored in deletedBlock.db of scm.

View File

@ -128,7 +128,7 @@ private void createToDeleteBlocks(ContainerManager mgr,
int numOfChunksPerBlock, File chunkDir) throws IOException {
for (int x = 0; x < numOfContainers; x++) {
String containerName = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName, conf);
ContainerData data = new ContainerData(containerName, new Long(x), conf);
mgr.createContainer(createSingleNodePipeline(containerName), data);
data = mgr.readContainer(containerName);
MetadataStore metadata = KeyUtils.getDB(data, conf);
@ -216,6 +216,7 @@ public void testBlockDeletion() throws Exception {
Assert.assertEquals(0, getUnderDeletionBlocksCount(meta));
svc.shutdown();
shutdownContainerMangaer(containerManager);
}
@Test
@ -244,6 +245,7 @@ public void testShutdownService() throws Exception {
// Shutdown service and verify all threads are stopped
service.shutdown();
GenericTestUtils.waitFor(() -> service.getThreadCount() == 0, 100, 1000);
shutdownContainerMangaer(containerManager);
}
@Test
@ -302,6 +304,7 @@ public void testBlockDeletionTimeout() throws Exception {
Assert.assertTrue(!newLog.getOutput().contains(
"Background task executes timed out, retrying in next interval"));
svc.shutdown();
shutdownContainerMangaer(containerManager);
}
@Test(timeout = 30000)
@ -333,6 +336,7 @@ public void testContainerThrottle() throws Exception {
Assert.assertEquals(10, chunksDir.listFiles().length);
} finally {
service.shutdown();
shutdownContainerMangaer(containerManager);
}
}
@ -376,6 +380,17 @@ public void testBlockThrottle() throws Exception {
Assert.assertEquals(0, chunksDir.listFiles().length);
} finally {
service.shutdown();
shutdownContainerMangaer(containerManager);
}
}
private void shutdownContainerMangaer(ContainerManager mgr)
throws IOException {
mgr.writeLock();
try {
mgr.shutdown();
} finally {
mgr.writeUnlock();
}
}
}

View File

@ -94,7 +94,7 @@ public void testRandomChoosingPolicy() throws IOException {
int numContainers = 10;
for (int i = 0; i < numContainers; i++) {
String containerName = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName, conf);
ContainerData data = new ContainerData(containerName, new Long(i), conf);
containerManager.createContainer(createSingleNodePipeline(containerName),
data);
Assert.assertTrue(
@ -144,7 +144,7 @@ public void testTopNOrderedChoosingPolicy() throws IOException {
// create [numContainers + 1] containers
for (int i = 0; i <= numContainers; i++) {
String containerName = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName, conf);
ContainerData data = new ContainerData(containerName, new Long(i), conf);
containerManager.createContainer(createSingleNodePipeline(containerName),
data);
Assert.assertTrue(

View File

@ -101,6 +101,7 @@ public class TestContainerPersistence {
private static KeyManagerImpl keyManager;
private static OzoneConfiguration conf;
private static List<StorageLocation> pathLists = new LinkedList<>();
private Long containerID = 8888L;;
@BeforeClass
public static void init() throws Throwable {
@ -180,7 +181,7 @@ public void cleanupDir() throws IOException {
public void testCreateContainer() throws Exception {
String containerName = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName, conf);
ContainerData data = new ContainerData(containerName, containerID++, conf);
data.addMetadata("VOLUME", "shire");
data.addMetadata("owner)", "bilbo");
containerManager.createContainer(createSingleNodePipeline(containerName),
@ -216,7 +217,7 @@ public void testCreateContainer() throws Exception {
public void testCreateDuplicateContainer() throws Exception {
String containerName = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName, conf);
ContainerData data = new ContainerData(containerName, containerID++, conf);
data.addMetadata("VOLUME", "shire");
data.addMetadata("owner)", "bilbo");
containerManager.createContainer(createSingleNodePipeline(containerName),
@ -236,14 +237,14 @@ public void testDeleteContainer() throws Exception {
String containerName2 = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName1, conf);
ContainerData data = new ContainerData(containerName1, containerID++, conf);
data.addMetadata("VOLUME", "shire");
data.addMetadata("owner)", "bilbo");
containerManager.createContainer(createSingleNodePipeline(containerName1),
data);
containerManager.closeContainer(containerName1);
data = new ContainerData(containerName2, conf);
data = new ContainerData(containerName2, containerID++, conf);
data.addMetadata("VOLUME", "shire");
data.addMetadata("owner)", "bilbo");
containerManager.createContainer(createSingleNodePipeline(containerName2),
@ -263,7 +264,7 @@ public void testDeleteContainer() throws Exception {
// Let us make sure that we are able to re-use a container name after
// delete.
data = new ContainerData(containerName1, conf);
data = new ContainerData(containerName1, containerID++, conf);
data.addMetadata("VOLUME", "shire");
data.addMetadata("owner)", "bilbo");
containerManager.createContainer(createSingleNodePipeline(containerName1),
@ -301,7 +302,8 @@ public void testGetContainerReports() throws Exception{
for (int i = 0; i < count; i++) {
String containerName = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName, conf);
ContainerData data = new ContainerData(containerName, containerID++,
conf);
containerManager.createContainer(createSingleNodePipeline(containerName),
data);
@ -337,8 +339,8 @@ public void testListContainer() throws IOException {
Map<String, ContainerData> testMap = new HashMap<>();
for (int x = 0; x < count; x++) {
String containerName = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName, conf);
ContainerData data = new ContainerData(containerName, containerID++,
conf);
data.addMetadata("VOLUME", "shire");
data.addMetadata("owner)", "bilbo");
containerManager.createContainer(createSingleNodePipeline(containerName),
@ -373,7 +375,7 @@ private ChunkInfo writeChunkHelper(String containerName, String keyName,
final int datalen = 1024;
Pipeline newPipeline =
new Pipeline(containerName, pipeline.getPipelineChannel());
ContainerData cData = new ContainerData(containerName, conf);
ContainerData cData = new ContainerData(containerName, containerID++, conf);
cData.addMetadata("VOLUME", "shire");
cData.addMetadata("owner", "bilbo");
if(!containerManager.getContainerMap()
@ -421,7 +423,7 @@ public void testWritReadManyChunks() throws IOException,
Pipeline pipeline = createSingleNodePipeline(containerName);
Map<String, ChunkInfo> fileHashMap = new HashMap<>();
ContainerData cData = new ContainerData(containerName, conf);
ContainerData cData = new ContainerData(containerName, containerID++, conf);
cData.addMetadata("VOLUME", "shire");
cData.addMetadata("owner)", "bilbo");
containerManager.createContainer(pipeline, cData);
@ -484,7 +486,7 @@ public void testPartialRead() throws Exception {
String keyName = OzoneUtils.getRequestID();
Pipeline pipeline = createSingleNodePipeline(containerName);
ContainerData cData = new ContainerData(containerName, conf);
ContainerData cData = new ContainerData(containerName, containerID++, conf);
cData.addMetadata("VOLUME", "shire");
cData.addMetadata("owner)", "bilbo");
containerManager.createContainer(pipeline, cData);
@ -518,7 +520,7 @@ public void testOverWrite() throws IOException,
String keyName = OzoneUtils.getRequestID();
Pipeline pipeline = createSingleNodePipeline(containerName);
ContainerData cData = new ContainerData(containerName, conf);
ContainerData cData = new ContainerData(containerName, containerID++, conf);
cData.addMetadata("VOLUME", "shire");
cData.addMetadata("owner)", "bilbo");
containerManager.createContainer(pipeline, cData);
@ -560,7 +562,7 @@ public void testMultipleWriteSingleRead() throws IOException,
String keyName = OzoneUtils.getRequestID();
Pipeline pipeline = createSingleNodePipeline(containerName);
ContainerData cData = new ContainerData(containerName, conf);
ContainerData cData = new ContainerData(containerName, containerID++, conf);
cData.addMetadata("VOLUME", "shire");
cData.addMetadata("owner)", "bilbo");
containerManager.createContainer(pipeline, cData);
@ -598,7 +600,7 @@ public void testDeleteChunk() throws IOException,
String keyName = OzoneUtils.getRequestID();
Pipeline pipeline = createSingleNodePipeline(containerName);
ContainerData cData = new ContainerData(containerName, conf);
ContainerData cData = new ContainerData(containerName, containerID++, conf);
cData.addMetadata("VOLUME", "shire");
cData.addMetadata("owner)", "bilbo");
containerManager.createContainer(pipeline, cData);
@ -744,7 +746,7 @@ public void testDeleteKeyTwice() throws IOException,
@Test
public void testUpdateContainer() throws IOException {
String containerName = OzoneUtils.getRequestID();
ContainerData data = new ContainerData(containerName, conf);
ContainerData data = new ContainerData(containerName, containerID++, conf);
data.addMetadata("VOLUME", "shire");
data.addMetadata("owner", "bilbo");
@ -755,7 +757,8 @@ public void testUpdateContainer() throws IOException {
File orgContainerFile = containerManager.getContainerFile(data);
Assert.assertTrue(orgContainerFile.exists());
ContainerData newData = new ContainerData(containerName, conf);
ContainerData newData = new ContainerData(containerName, containerID++,
conf);
newData.addMetadata("VOLUME", "shire_new");
newData.addMetadata("owner", "bilbo_new");
@ -807,7 +810,7 @@ public void testUpdateContainer() throws IOException {
}
// Update with force flag, it should be success.
newData = new ContainerData(containerName, conf);
newData = new ContainerData(containerName, containerID++, conf);
newData.addMetadata("VOLUME", "shire_new_1");
newData.addMetadata("owner", "bilbo_new_1");
containerManager.updateContainer(createSingleNodePipeline(containerName),

View File

@ -0,0 +1,138 @@
/**
* 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.ksm;
import org.apache.commons.lang.RandomStringUtils;
import org.apache.hadoop.conf.OzoneConfiguration;
import org.apache.hadoop.ozone.MiniOzoneClassicCluster;
import org.apache.hadoop.ozone.OzoneConfigKeys;
import org.apache.hadoop.ozone.OzoneConsts;
import org.apache.hadoop.ozone.client.*;
import org.apache.hadoop.ozone.client.io.OzoneOutputStream;
import org.apache.hadoop.ozone.container.common.helpers.ContainerData;
import org.apache.hadoop.ozone.ksm.helpers.KsmKeyArgs;
import org.apache.hadoop.ozone.ksm.helpers.KsmKeyLocationInfo;
import org.apache.hadoop.ozone.protocol.proto.OzoneProtos;
import org.apache.hadoop.ozone.scm.StorageContainerManager;
import org.apache.hadoop.scm.container.common.helpers.ContainerInfo;
import org.apache.hadoop.scm.container.common.helpers.StorageContainerException;
import org.junit.AfterClass;
import org.junit.BeforeClass;
import org.junit.Rule;
import org.junit.Test;
import org.junit.rules.ExpectedException;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import java.io.IOException;
/**
* This class tests container report with DN container state info.
*/
public class TestContainerReportWithKeys {
private static final Logger LOG = LoggerFactory.getLogger(
TestContainerReportWithKeys.class);
private static MiniOzoneClassicCluster cluster = null;
private static OzoneConfiguration conf;
private static StorageContainerManager scm;
@Rule
public ExpectedException exception = ExpectedException.none();
/**
* Create a MiniDFSCluster for testing.
* <p>
* Ozone is made active by setting OZONE_ENABLED = true and
* OZONE_HANDLER_TYPE_KEY = "distributed"
*
* @throws IOException
*/
@BeforeClass
public static void init() throws Exception {
conf = new OzoneConfiguration();
conf.set(OzoneConfigKeys.OZONE_HANDLER_TYPE_KEY,
OzoneConsts.OZONE_HANDLER_DISTRIBUTED);
cluster = new MiniOzoneClassicCluster.Builder(conf)
.setHandlerType(OzoneConsts.OZONE_HANDLER_DISTRIBUTED).build();
scm = cluster.getStorageContainerManager();
}
/**
* Shutdown MiniDFSCluster.
*/
@AfterClass
public static void shutdown() {
if (cluster != null) {
cluster.shutdown();
}
}
@Test
public void testContainerReportKeyWrite() throws Exception {
final String volumeName = "volume" + RandomStringUtils.randomNumeric(5);
final String bucketName = "bucket" + RandomStringUtils.randomNumeric(5);
final String keyName = "key" + RandomStringUtils.randomNumeric(5);
final int keySize = 100;
OzoneClient client = OzoneClientFactory.getClient(conf);
ObjectStore objectStore = client.getObjectStore();
objectStore.createVolume(volumeName);
objectStore.getVolume(volumeName).createBucket(bucketName);
OzoneOutputStream key = objectStore.getVolume(volumeName).getBucket(bucketName)
.createKey(keyName, keySize, ReplicationType.STAND_ALONE,
ReplicationFactor.ONE);
String dataString = RandomStringUtils.randomAlphabetic(keySize);
key.write(dataString.getBytes());
key.close();
KsmKeyArgs keyArgs = new KsmKeyArgs.Builder()
.setVolumeName(volumeName)
.setBucketName(bucketName)
.setKeyName(keyName)
.setType(OzoneProtos.ReplicationType.STAND_ALONE)
.setFactor(OzoneProtos.ReplicationFactor.ONE).setDataSize(keySize)
.build();
KsmKeyLocationInfo keyInfo =
cluster.getKeySpaceManager().lookupKey(keyArgs).getKeyLocationVersions()
.get(0).getBlocksLatestVersionOnly().get(0);
ContainerData cd = getContainerData(cluster, keyInfo.getContainerName());
LOG.info("DN Container Data: keyCount: {} used: {} ",
cd.getKeyCount(), cd.getBytesUsed());
ContainerInfo cinfo = scm.getContainerInfo(keyInfo.getContainerName());
LOG.info("SCM Container Info keyCount: {} usedBytes: {}",
cinfo.getNumberOfKeys(), cinfo.getUsedBytes());
}
private static ContainerData getContainerData(MiniOzoneClassicCluster clus,
String containerName) {
ContainerData containerData = null;
try {
containerData = clus.getDataNodes().get(0).getOzoneContainerManager()
.getContainerManager().readContainer(containerName);
} catch (StorageContainerException e) {
throw new AssertionError(e);
}
return containerData;
}
}