You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@ozone.apache.org by GitBox <gi...@apache.org> on 2020/01/29 19:07:25 UTC

[GitHub] [hadoop-ozone] avijayanhwx opened a new pull request #503: HDDS-2850. Handle Create container use case in Recon.

avijayanhwx opened a new pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503
 
 
   ## What changes were proposed in this pull request?
   CREATE container needs to be handled differently in Recon since these actions are initiated in the SCM, and Recon does not know about that. It should not throw ContainerNotFoundException when it suddenly sees a new container. The idea is to let Recon ask SCM about the new container whenever it sees a new one.
   
   ## What is the link to the Apache JIRA
   https://issues.apache.org/jira/browse/HDDS-2850
   
   ## How was this patch tested?
   Manually tested.
   Unit tested.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373151740
 
 

 ##########
 File path: hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/scm/TestReconIncrementalContainerReportHandler.java
 ##########
 @@ -0,0 +1,159 @@
+/**
+ * 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.recon.scm;
+
+import static org.apache.hadoop.hdds.protocol.MockDatanodeDetails.randomDatanodeDetails;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.LifeCycleState.OPEN;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor.ONE;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType.STAND_ALONE;
+import static org.apache.hadoop.hdds.scm.ScmConfigKeys.OZONE_SCM_NAMES;
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_METADATA_DIRS;
+import static org.apache.hadoop.ozone.recon.AbstractOMMetadataManagerTest.getRandomPipeline;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertTrue;
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.times;
+import static org.mockito.Mockito.verify;
+import static org.mockito.Mockito.when;
+
+import java.io.IOException;
+
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto.State;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.IncrementalContainerReportProto;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerInfo;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.net.NetworkTopology;
+import org.apache.hadoop.hdds.scm.net.NetworkTopologyImpl;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.SCMNodeManager;
+import org.apache.hadoop.hdds.scm.node.states.NodeNotFoundException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.server.SCMDatanodeHeartbeatDispatcher.IncrementalContainerReportFromDatanode;
+import org.apache.hadoop.hdds.scm.server.SCMStorageConfig;
+import org.apache.hadoop.hdds.server.events.EventPublisher;
+import org.apache.hadoop.hdds.server.events.EventQueue;
+import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+
+/**
+ * Test Recon ICR handler.
+ */
+public class TestReconIncrementalContainerReportHandler {
+
+  @Rule
+  public TemporaryFolder temporaryFolder = new TemporaryFolder();
+
+  private OzoneConfiguration conf;
+  private SCMStorageConfig scmStorageConfig;
+  private ReconPipelineManager pipelineManager;
+  private ReconContainerManager containerManager;
+
+  @Before
+  public void setUp() throws Exception {
+    conf = new OzoneConfiguration();
+    conf.set(OZONE_METADATA_DIRS,
+        temporaryFolder.newFolder().getAbsolutePath());
+    conf.set(OZONE_SCM_NAMES, "localhost");
+    scmStorageConfig = new ReconStorageConfig(conf);
+    NetworkTopology clusterMap = new NetworkTopologyImpl(conf);
+    EventQueue eventQueue = new EventQueue();
+    NodeManager nodeManager =
+        new SCMNodeManager(conf, scmStorageConfig, eventQueue, clusterMap);
+    pipelineManager = new ReconPipelineManager(conf, nodeManager, eventQueue);
+    containerManager = new ReconContainerManager(conf, pipelineManager);
+  }
+
+  @After
+  public void tearDown() throws IOException {
+    containerManager.close();
+    pipelineManager.close();
+  }
 
 Review comment:
   Thanks for keeping me honest :) Fixed.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r372784812
 
 

 ##########
 File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/SCMContainerManager.java
 ##########
 @@ -194,6 +194,18 @@ public ContainerInfo getContainer(final ContainerID containerID)
     return containerStateManager.getContainer(containerID);
   }
 
+  @Override
+  public boolean exists(ContainerID containerID) {
+    lock.lock();
 
 Review comment:
   Not sure. SCM ContainerManager has had a Reentrant lock from the first. I will investigate why it is not a RW lock, and create a new JIRA for this.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373151646
 
 

 ##########
 File path: hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/scm/TestReconContainerManager.java
 ##########
 @@ -0,0 +1,120 @@
+/**
+ * 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.recon.scm;
+
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.LifeCycleState.OPEN;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor.ONE;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType.STAND_ALONE;
+import static org.apache.hadoop.hdds.scm.ScmConfigKeys.OZONE_SCM_NAMES;
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_METADATA_DIRS;
+import static org.apache.hadoop.ozone.recon.AbstractOMMetadataManagerTest.getRandomPipeline;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
+
+import java.io.IOException;
+import java.util.List;
+import java.util.NavigableSet;
+
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerInfo;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.net.NetworkTopology;
+import org.apache.hadoop.hdds.scm.net.NetworkTopologyImpl;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.SCMNodeManager;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.server.SCMStorageConfig;
+import org.apache.hadoop.hdds.server.events.EventQueue;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+
+/**
+ * Test Recon Container Manager.
+ */
+public class TestReconContainerManager {
+
+  @Rule
+  public TemporaryFolder temporaryFolder = new TemporaryFolder();
+
+  private OzoneConfiguration conf;
+  private SCMStorageConfig scmStorageConfig;
+  private ReconPipelineManager pipelineManager;
+  private ReconContainerManager containerManager;
+
+  @Before
+  public void setUp() throws Exception {
+    conf = new OzoneConfiguration();
+    conf.set(OZONE_METADATA_DIRS,
+        temporaryFolder.newFolder().getAbsolutePath());
+    conf.set(OZONE_SCM_NAMES, "localhost");
+    scmStorageConfig = new ReconStorageConfig(conf);
+    NetworkTopology clusterMap = new NetworkTopologyImpl(conf);
+    EventQueue eventQueue = new EventQueue();
+    NodeManager nodeManager =
+        new SCMNodeManager(conf, scmStorageConfig, eventQueue, clusterMap);
+    pipelineManager = new ReconPipelineManager(conf, nodeManager, eventQueue);
+    containerManager = new ReconContainerManager(conf, pipelineManager);
+  }
+
+  @After
+  public void tearDown() throws IOException {
+    containerManager.close();
+    pipelineManager.close();
+  }
+
+  @Test
+  public void testAddNewContainer() throws IOException {
+    ContainerID containerID = new ContainerID(100L);
+    Pipeline pipeline = getRandomPipeline();
+    pipelineManager.addPipeline(pipeline);
+    ContainerInfo containerInfo =
+        new ContainerInfo.Builder()
+            .setContainerID(containerID.getId())
+            .setNumberOfKeys(10)
+            .setPipelineID(pipeline.getId())
+            .setReplicationFactor(ONE)
+            .setOwner("test")
+            .setState(OPEN)
+            .setReplicationType(STAND_ALONE)
+            .build();
+    ContainerWithPipeline containerWithPipeline =
+        new ContainerWithPipeline(containerInfo, pipeline);
+
+    assertFalse(containerManager.exists(containerID));
+
+    containerManager.addNewContainer(
+        containerID.getId(), containerWithPipeline);
+
+    assertTrue(containerManager.exists(containerID));
+
+    List<ContainerInfo> containers = containerManager.getContainers(OPEN);
+    assertEquals(1, containers.size());
+    assertTrue(containers.get(0).equals(containerInfo));
+    NavigableSet<ContainerID> containersInPipeline =
+        pipelineManager.getContainersInPipeline(pipeline.getId());
+    assertEquals(1, containersInPipeline.size());
+    assertTrue(containersInPipeline.first().equals(containerID));
 
 Review comment:
   Fixed.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r372785181
 
 

 ##########
 File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconIncrementalContainerReportHandler.java
 ##########
 @@ -0,0 +1,108 @@
+/**
+ * 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.recon.scm;
+
+import java.io.IOException;
+
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerManager;
+import org.apache.hadoop.hdds.scm.container.ContainerNotFoundException;
+import org.apache.hadoop.hdds.scm.container.IncrementalContainerReportHandler;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.states.NodeNotFoundException;
+import org.apache.hadoop.hdds.scm.server.SCMDatanodeHeartbeatDispatcher.IncrementalContainerReportFromDatanode;
+import org.apache.hadoop.hdds.server.events.EventPublisher;
+import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Recon ICR handler.
+ */
+public class ReconIncrementalContainerReportHandler
+    extends IncrementalContainerReportHandler {
+
+  private static final Logger LOG = LoggerFactory.getLogger(
+      ReconIncrementalContainerReportHandler.class);
+
+  private StorageContainerServiceProvider scmClient;
+
+  public ReconIncrementalContainerReportHandler(NodeManager nodeManager,
+      ContainerManager containerManager,
+      StorageContainerServiceProvider scmClient) {
+    super(nodeManager, containerManager);
+    this.scmClient = scmClient;
+  }
+
+  @Override
+  public void onMessage(final IncrementalContainerReportFromDatanode report,
+                        final EventPublisher publisher) {
+    if (LOG.isDebugEnabled()) {
+      LOG.debug("Processing incremental container report from data node {}",
+          report.getDatanodeDetails().getUuid());
 
 Review comment:
   Thanks for the suggestion, fixed!

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] avijayanhwx commented on issue #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
avijayanhwx commented on issue #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#issuecomment-580418643
 
 
   @adoroszlai Thanks for the review. Addressed your review comments.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] swagle commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
swagle commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r372725292
 
 

 ##########
 File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconIncrementalContainerReportHandler.java
 ##########
 @@ -0,0 +1,108 @@
+/**
+ * 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.recon.scm;
+
+import java.io.IOException;
+
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerManager;
+import org.apache.hadoop.hdds.scm.container.ContainerNotFoundException;
+import org.apache.hadoop.hdds.scm.container.IncrementalContainerReportHandler;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.states.NodeNotFoundException;
+import org.apache.hadoop.hdds.scm.server.SCMDatanodeHeartbeatDispatcher.IncrementalContainerReportFromDatanode;
+import org.apache.hadoop.hdds.server.events.EventPublisher;
+import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Recon ICR handler.
+ */
+public class ReconIncrementalContainerReportHandler
+    extends IncrementalContainerReportHandler {
+
+  private static final Logger LOG = LoggerFactory.getLogger(
+      ReconIncrementalContainerReportHandler.class);
+
+  private StorageContainerServiceProvider scmClient;
+
+  public ReconIncrementalContainerReportHandler(NodeManager nodeManager,
+      ContainerManager containerManager,
+      StorageContainerServiceProvider scmClient) {
+    super(nodeManager, containerManager);
+    this.scmClient = scmClient;
+  }
+
+  @Override
+  public void onMessage(final IncrementalContainerReportFromDatanode report,
+                        final EventPublisher publisher) {
+    if (LOG.isDebugEnabled()) {
+      LOG.debug("Processing incremental container report from data node {}",
+          report.getDatanodeDetails().getUuid());
 
 Review comment:
   Minor: instead of getUuid() let toString method of DatanodeDetails determine what is printed.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] adoroszlai commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
adoroszlai commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373131144
 
 

 ##########
 File path: hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/scm/TestReconContainerManager.java
 ##########
 @@ -0,0 +1,120 @@
+/**
+ * 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.recon.scm;
+
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.LifeCycleState.OPEN;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor.ONE;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType.STAND_ALONE;
+import static org.apache.hadoop.hdds.scm.ScmConfigKeys.OZONE_SCM_NAMES;
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_METADATA_DIRS;
+import static org.apache.hadoop.ozone.recon.AbstractOMMetadataManagerTest.getRandomPipeline;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
+
+import java.io.IOException;
+import java.util.List;
+import java.util.NavigableSet;
+
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerInfo;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.net.NetworkTopology;
+import org.apache.hadoop.hdds.scm.net.NetworkTopologyImpl;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.SCMNodeManager;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.server.SCMStorageConfig;
+import org.apache.hadoop.hdds.server.events.EventQueue;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+
+/**
+ * Test Recon Container Manager.
+ */
+public class TestReconContainerManager {
+
+  @Rule
+  public TemporaryFolder temporaryFolder = new TemporaryFolder();
+
+  private OzoneConfiguration conf;
+  private SCMStorageConfig scmStorageConfig;
+  private ReconPipelineManager pipelineManager;
+  private ReconContainerManager containerManager;
+
+  @Before
+  public void setUp() throws Exception {
+    conf = new OzoneConfiguration();
+    conf.set(OZONE_METADATA_DIRS,
+        temporaryFolder.newFolder().getAbsolutePath());
+    conf.set(OZONE_SCM_NAMES, "localhost");
+    scmStorageConfig = new ReconStorageConfig(conf);
+    NetworkTopology clusterMap = new NetworkTopologyImpl(conf);
+    EventQueue eventQueue = new EventQueue();
+    NodeManager nodeManager =
+        new SCMNodeManager(conf, scmStorageConfig, eventQueue, clusterMap);
+    pipelineManager = new ReconPipelineManager(conf, nodeManager, eventQueue);
+    containerManager = new ReconContainerManager(conf, pipelineManager);
+  }
+
+  @After
+  public void tearDown() throws IOException {
+    containerManager.close();
+    pipelineManager.close();
+  }
+
+  @Test
+  public void testAddNewContainer() throws IOException {
+    ContainerID containerID = new ContainerID(100L);
+    Pipeline pipeline = getRandomPipeline();
+    pipelineManager.addPipeline(pipeline);
+    ContainerInfo containerInfo =
+        new ContainerInfo.Builder()
+            .setContainerID(containerID.getId())
+            .setNumberOfKeys(10)
+            .setPipelineID(pipeline.getId())
+            .setReplicationFactor(ONE)
+            .setOwner("test")
+            .setState(OPEN)
+            .setReplicationType(STAND_ALONE)
+            .build();
+    ContainerWithPipeline containerWithPipeline =
+        new ContainerWithPipeline(containerInfo, pipeline);
+
+    assertFalse(containerManager.exists(containerID));
+
+    containerManager.addNewContainer(
+        containerID.getId(), containerWithPipeline);
+
+    assertTrue(containerManager.exists(containerID));
+
+    List<ContainerInfo> containers = containerManager.getContainers(OPEN);
+    assertEquals(1, containers.size());
+    assertTrue(containers.get(0).equals(containerInfo));
+    NavigableSet<ContainerID> containersInPipeline =
+        pipelineManager.getContainersInPipeline(pipeline.getId());
+    assertEquals(1, containersInPipeline.size());
+    assertTrue(containersInPipeline.first().equals(containerID));
 
 Review comment:
   ```suggestion
       assertEquals(containerID, containersInPipeline.first());
   ```

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373151613
 
 

 ##########
 File path: hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/scm/TestReconContainerManager.java
 ##########
 @@ -0,0 +1,120 @@
+/**
+ * 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.recon.scm;
+
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.LifeCycleState.OPEN;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor.ONE;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType.STAND_ALONE;
+import static org.apache.hadoop.hdds.scm.ScmConfigKeys.OZONE_SCM_NAMES;
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_METADATA_DIRS;
+import static org.apache.hadoop.ozone.recon.AbstractOMMetadataManagerTest.getRandomPipeline;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
+
+import java.io.IOException;
+import java.util.List;
+import java.util.NavigableSet;
+
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerInfo;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.net.NetworkTopology;
+import org.apache.hadoop.hdds.scm.net.NetworkTopologyImpl;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.SCMNodeManager;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.server.SCMStorageConfig;
+import org.apache.hadoop.hdds.server.events.EventQueue;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+
+/**
+ * Test Recon Container Manager.
+ */
+public class TestReconContainerManager {
+
+  @Rule
+  public TemporaryFolder temporaryFolder = new TemporaryFolder();
+
+  private OzoneConfiguration conf;
+  private SCMStorageConfig scmStorageConfig;
+  private ReconPipelineManager pipelineManager;
+  private ReconContainerManager containerManager;
+
+  @Before
+  public void setUp() throws Exception {
+    conf = new OzoneConfiguration();
+    conf.set(OZONE_METADATA_DIRS,
+        temporaryFolder.newFolder().getAbsolutePath());
+    conf.set(OZONE_SCM_NAMES, "localhost");
+    scmStorageConfig = new ReconStorageConfig(conf);
+    NetworkTopology clusterMap = new NetworkTopologyImpl(conf);
+    EventQueue eventQueue = new EventQueue();
+    NodeManager nodeManager =
+        new SCMNodeManager(conf, scmStorageConfig, eventQueue, clusterMap);
+    pipelineManager = new ReconPipelineManager(conf, nodeManager, eventQueue);
+    containerManager = new ReconContainerManager(conf, pipelineManager);
+  }
+
+  @After
+  public void tearDown() throws IOException {
+    containerManager.close();
+    pipelineManager.close();
+  }
+
+  @Test
+  public void testAddNewContainer() throws IOException {
+    ContainerID containerID = new ContainerID(100L);
+    Pipeline pipeline = getRandomPipeline();
+    pipelineManager.addPipeline(pipeline);
+    ContainerInfo containerInfo =
+        new ContainerInfo.Builder()
+            .setContainerID(containerID.getId())
+            .setNumberOfKeys(10)
+            .setPipelineID(pipeline.getId())
+            .setReplicationFactor(ONE)
+            .setOwner("test")
+            .setState(OPEN)
+            .setReplicationType(STAND_ALONE)
+            .build();
+    ContainerWithPipeline containerWithPipeline =
+        new ContainerWithPipeline(containerInfo, pipeline);
+
+    assertFalse(containerManager.exists(containerID));
+
+    containerManager.addNewContainer(
+        containerID.getId(), containerWithPipeline);
+
+    assertTrue(containerManager.exists(containerID));
+
+    List<ContainerInfo> containers = containerManager.getContainers(OPEN);
+    assertEquals(1, containers.size());
+    assertTrue(containers.get(0).equals(containerInfo));
 
 Review comment:
   Fixed.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] nandakumar131 merged pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
nandakumar131 merged pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503
 
 
   

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373154420
 
 

 ##########
 File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconIncrementalContainerReportHandler.java
 ##########
 @@ -0,0 +1,108 @@
+/**
+ * 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.recon.scm;
+
+import java.io.IOException;
+
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerManager;
+import org.apache.hadoop.hdds.scm.container.ContainerNotFoundException;
+import org.apache.hadoop.hdds.scm.container.IncrementalContainerReportHandler;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.states.NodeNotFoundException;
+import org.apache.hadoop.hdds.scm.server.SCMDatanodeHeartbeatDispatcher.IncrementalContainerReportFromDatanode;
+import org.apache.hadoop.hdds.server.events.EventPublisher;
+import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Recon ICR handler.
+ */
+public class ReconIncrementalContainerReportHandler
+    extends IncrementalContainerReportHandler {
+
+  private static final Logger LOG = LoggerFactory.getLogger(
+      ReconIncrementalContainerReportHandler.class);
+
+  private StorageContainerServiceProvider scmClient;
+
+  public ReconIncrementalContainerReportHandler(NodeManager nodeManager,
+      ContainerManager containerManager,
+      StorageContainerServiceProvider scmClient) {
+    super(nodeManager, containerManager);
+    this.scmClient = scmClient;
+  }
+
+  @Override
+  public void onMessage(final IncrementalContainerReportFromDatanode report,
+                        final EventPublisher publisher) {
+    if (LOG.isDebugEnabled()) {
+      LOG.debug("Processing incremental container report from data node {}",
+          report.getDatanodeDetails());
+    }
+
+    ReconContainerManager containerManager =
+        (ReconContainerManager) getContainerManager();
+    boolean success = true;
+    for (ContainerReplicaProto replicaProto :
+        report.getReport().getReportList()) {
+      try {
+        final DatanodeDetails dd = report.getDatanodeDetails();
+        final ContainerID id = ContainerID.valueof(
+            replicaProto.getContainerID());
+        if (!getContainerManager().exists(id)) {
+          LOG.info("New container {} got from {}.", id,
+              report.getDatanodeDetails());
+          try {
+            ContainerWithPipeline containerWithPipeline =
+                scmClient.getContainerWithPipeline(id.getId());
+            containerManager.addNewContainer(id.getId(), containerWithPipeline);
+          } catch (IOException ioEx) {
+            LOG.error("Exception while getting new container info from SCM",
+                ioEx);
+            return;
+          }
+        }
+        getNodeManager().addContainer(dd, id);
+        processContainerReplica(dd, replicaProto);
+      } catch (ContainerNotFoundException e) {
+        success = false;
+        LOG.warn("Container {} not found!", replicaProto.getContainerID());
+      } catch (NodeNotFoundException ex) {
+        success = false;
+        LOG.error("Received ICR from unknown datanode {} {}",
+            report.getDatanodeDetails(), ex);
+      } catch (IOException e) {
+        success = false;
+        LOG.error("Exception while processing ICR for container {}",
+            replicaProto.getContainerID());
+      }
+    }
+
+    if (success) {
+      getContainerManager().notifyContainerReportProcessing(false, true);
+    } else {
+      getContainerManager().notifyContainerReportProcessing(false, false);
+    }
 
 Review comment:
   This code was just copied from SCM's IncrementalContainerReportHandler. FYI -> https://github.com/apache/hadoop/pull/1541#discussion_r329256932

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] adoroszlai commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
adoroszlai commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373135520
 
 

 ##########
 File path: hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/scm/TestReconIncrementalContainerReportHandler.java
 ##########
 @@ -0,0 +1,159 @@
+/**
+ * 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.recon.scm;
+
+import static org.apache.hadoop.hdds.protocol.MockDatanodeDetails.randomDatanodeDetails;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.LifeCycleState.OPEN;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor.ONE;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType.STAND_ALONE;
+import static org.apache.hadoop.hdds.scm.ScmConfigKeys.OZONE_SCM_NAMES;
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_METADATA_DIRS;
+import static org.apache.hadoop.ozone.recon.AbstractOMMetadataManagerTest.getRandomPipeline;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertTrue;
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.times;
+import static org.mockito.Mockito.verify;
+import static org.mockito.Mockito.when;
+
+import java.io.IOException;
+
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto.State;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.IncrementalContainerReportProto;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerInfo;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.net.NetworkTopology;
+import org.apache.hadoop.hdds.scm.net.NetworkTopologyImpl;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.SCMNodeManager;
+import org.apache.hadoop.hdds.scm.node.states.NodeNotFoundException;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.server.SCMDatanodeHeartbeatDispatcher.IncrementalContainerReportFromDatanode;
+import org.apache.hadoop.hdds.scm.server.SCMStorageConfig;
+import org.apache.hadoop.hdds.server.events.EventPublisher;
+import org.apache.hadoop.hdds.server.events.EventQueue;
+import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+
+/**
+ * Test Recon ICR handler.
+ */
+public class TestReconIncrementalContainerReportHandler {
+
+  @Rule
+  public TemporaryFolder temporaryFolder = new TemporaryFolder();
+
+  private OzoneConfiguration conf;
+  private SCMStorageConfig scmStorageConfig;
+  private ReconPipelineManager pipelineManager;
+  private ReconContainerManager containerManager;
+
+  @Before
+  public void setUp() throws Exception {
+    conf = new OzoneConfiguration();
+    conf.set(OZONE_METADATA_DIRS,
+        temporaryFolder.newFolder().getAbsolutePath());
+    conf.set(OZONE_SCM_NAMES, "localhost");
+    scmStorageConfig = new ReconStorageConfig(conf);
+    NetworkTopology clusterMap = new NetworkTopologyImpl(conf);
+    EventQueue eventQueue = new EventQueue();
+    NodeManager nodeManager =
+        new SCMNodeManager(conf, scmStorageConfig, eventQueue, clusterMap);
+    pipelineManager = new ReconPipelineManager(conf, nodeManager, eventQueue);
+    containerManager = new ReconContainerManager(conf, pipelineManager);
+  }
+
+  @After
+  public void tearDown() throws IOException {
+    containerManager.close();
+    pipelineManager.close();
+  }
 
 Review comment:
   Seems to be duplicated between these test classes.  Would be nice to reduce it in a follow-up.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] swagle commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
swagle commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r372723418
 
 

 ##########
 File path: hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/container/SCMContainerManager.java
 ##########
 @@ -194,6 +194,18 @@ public ContainerInfo getContainer(final ContainerID containerID)
     return containerStateManager.getContainer(containerID);
   }
 
+  @Override
+  public boolean exists(ContainerID containerID) {
+    lock.lock();
 
 Review comment:
   This should likely be filed as separate Jira, but why is this not a ReadWriteLock?

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] nandakumar131 commented on issue #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
nandakumar131 commented on issue #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#issuecomment-580688212
 
 
   @adoroszlai  Thanks for the review. I will take a look at the changes and merge it.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] adoroszlai commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
adoroszlai commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373127059
 
 

 ##########
 File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconIncrementalContainerReportHandler.java
 ##########
 @@ -0,0 +1,108 @@
+/**
+ * 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.recon.scm;
+
+import java.io.IOException;
+
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerManager;
+import org.apache.hadoop.hdds.scm.container.ContainerNotFoundException;
+import org.apache.hadoop.hdds.scm.container.IncrementalContainerReportHandler;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.states.NodeNotFoundException;
+import org.apache.hadoop.hdds.scm.server.SCMDatanodeHeartbeatDispatcher.IncrementalContainerReportFromDatanode;
+import org.apache.hadoop.hdds.server.events.EventPublisher;
+import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Recon ICR handler.
+ */
+public class ReconIncrementalContainerReportHandler
+    extends IncrementalContainerReportHandler {
+
+  private static final Logger LOG = LoggerFactory.getLogger(
+      ReconIncrementalContainerReportHandler.class);
+
+  private StorageContainerServiceProvider scmClient;
+
+  public ReconIncrementalContainerReportHandler(NodeManager nodeManager,
+      ContainerManager containerManager,
+      StorageContainerServiceProvider scmClient) {
+    super(nodeManager, containerManager);
+    this.scmClient = scmClient;
+  }
+
+  @Override
+  public void onMessage(final IncrementalContainerReportFromDatanode report,
+                        final EventPublisher publisher) {
+    if (LOG.isDebugEnabled()) {
+      LOG.debug("Processing incremental container report from data node {}",
+          report.getDatanodeDetails());
+    }
+
+    ReconContainerManager containerManager =
+        (ReconContainerManager) getContainerManager();
+    boolean success = true;
+    for (ContainerReplicaProto replicaProto :
+        report.getReport().getReportList()) {
+      try {
+        final DatanodeDetails dd = report.getDatanodeDetails();
+        final ContainerID id = ContainerID.valueof(
+            replicaProto.getContainerID());
+        if (!getContainerManager().exists(id)) {
+          LOG.info("New container {} got from {}.", id,
+              report.getDatanodeDetails());
+          try {
+            ContainerWithPipeline containerWithPipeline =
+                scmClient.getContainerWithPipeline(id.getId());
+            containerManager.addNewContainer(id.getId(), containerWithPipeline);
+          } catch (IOException ioEx) {
+            LOG.error("Exception while getting new container info from SCM",
+                ioEx);
+            return;
+          }
+        }
+        getNodeManager().addContainer(dd, id);
+        processContainerReplica(dd, replicaProto);
+      } catch (ContainerNotFoundException e) {
+        success = false;
+        LOG.warn("Container {} not found!", replicaProto.getContainerID());
+      } catch (NodeNotFoundException ex) {
+        success = false;
+        LOG.error("Received ICR from unknown datanode {} {}",
+            report.getDatanodeDetails(), ex);
+      } catch (IOException e) {
+        success = false;
+        LOG.error("Exception while processing ICR for container {}",
+            replicaProto.getContainerID());
+      }
+    }
+
+    if (success) {
+      getContainerManager().notifyContainerReportProcessing(false, true);
+    } else {
+      getContainerManager().notifyContainerReportProcessing(false, false);
+    }
 
 Review comment:
   Nit: can be simplified to
   
   ```
   getContainerManager().notifyContainerReportProcessing(false, success);
   ```

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
avijayanhwx commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373151565
 
 

 ##########
 File path: hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconIncrementalContainerReportHandler.java
 ##########
 @@ -0,0 +1,108 @@
+/**
+ * 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.recon.scm;
+
+import java.io.IOException;
+
+import org.apache.hadoop.hdds.protocol.DatanodeDetails;
+import org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerManager;
+import org.apache.hadoop.hdds.scm.container.ContainerNotFoundException;
+import org.apache.hadoop.hdds.scm.container.IncrementalContainerReportHandler;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.states.NodeNotFoundException;
+import org.apache.hadoop.hdds.scm.server.SCMDatanodeHeartbeatDispatcher.IncrementalContainerReportFromDatanode;
+import org.apache.hadoop.hdds.server.events.EventPublisher;
+import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Recon ICR handler.
+ */
+public class ReconIncrementalContainerReportHandler
+    extends IncrementalContainerReportHandler {
+
+  private static final Logger LOG = LoggerFactory.getLogger(
+      ReconIncrementalContainerReportHandler.class);
+
+  private StorageContainerServiceProvider scmClient;
+
+  public ReconIncrementalContainerReportHandler(NodeManager nodeManager,
+      ContainerManager containerManager,
+      StorageContainerServiceProvider scmClient) {
+    super(nodeManager, containerManager);
+    this.scmClient = scmClient;
+  }
+
+  @Override
+  public void onMessage(final IncrementalContainerReportFromDatanode report,
+                        final EventPublisher publisher) {
+    if (LOG.isDebugEnabled()) {
+      LOG.debug("Processing incremental container report from data node {}",
+          report.getDatanodeDetails());
+    }
+
+    ReconContainerManager containerManager =
+        (ReconContainerManager) getContainerManager();
+    boolean success = true;
+    for (ContainerReplicaProto replicaProto :
+        report.getReport().getReportList()) {
+      try {
+        final DatanodeDetails dd = report.getDatanodeDetails();
+        final ContainerID id = ContainerID.valueof(
+            replicaProto.getContainerID());
+        if (!getContainerManager().exists(id)) {
+          LOG.info("New container {} got from {}.", id,
+              report.getDatanodeDetails());
+          try {
+            ContainerWithPipeline containerWithPipeline =
+                scmClient.getContainerWithPipeline(id.getId());
+            containerManager.addNewContainer(id.getId(), containerWithPipeline);
+          } catch (IOException ioEx) {
+            LOG.error("Exception while getting new container info from SCM",
+                ioEx);
+            return;
+          }
+        }
+        getNodeManager().addContainer(dd, id);
+        processContainerReplica(dd, replicaProto);
+      } catch (ContainerNotFoundException e) {
+        success = false;
+        LOG.warn("Container {} not found!", replicaProto.getContainerID());
+      } catch (NodeNotFoundException ex) {
+        success = false;
+        LOG.error("Received ICR from unknown datanode {} {}",
+            report.getDatanodeDetails(), ex);
+      } catch (IOException e) {
+        success = false;
+        LOG.error("Exception while processing ICR for container {}",
+            replicaProto.getContainerID());
+      }
+    }
+
+    if (success) {
+      getContainerManager().notifyContainerReportProcessing(false, true);
+    } else {
+      getContainerManager().notifyContainerReportProcessing(false, false);
+    }
 
 Review comment:
   Fixed.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] adoroszlai commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
adoroszlai commented on a change in pull request #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#discussion_r373130838
 
 

 ##########
 File path: hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/scm/TestReconContainerManager.java
 ##########
 @@ -0,0 +1,120 @@
+/**
+ * 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.recon.scm;
+
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.LifeCycleState.OPEN;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor.ONE;
+import static org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType.STAND_ALONE;
+import static org.apache.hadoop.hdds.scm.ScmConfigKeys.OZONE_SCM_NAMES;
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_METADATA_DIRS;
+import static org.apache.hadoop.ozone.recon.AbstractOMMetadataManagerTest.getRandomPipeline;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
+
+import java.io.IOException;
+import java.util.List;
+import java.util.NavigableSet;
+
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.scm.container.ContainerID;
+import org.apache.hadoop.hdds.scm.container.ContainerInfo;
+import org.apache.hadoop.hdds.scm.container.common.helpers.ContainerWithPipeline;
+import org.apache.hadoop.hdds.scm.net.NetworkTopology;
+import org.apache.hadoop.hdds.scm.net.NetworkTopologyImpl;
+import org.apache.hadoop.hdds.scm.node.NodeManager;
+import org.apache.hadoop.hdds.scm.node.SCMNodeManager;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.server.SCMStorageConfig;
+import org.apache.hadoop.hdds.server.events.EventQueue;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+
+/**
+ * Test Recon Container Manager.
+ */
+public class TestReconContainerManager {
+
+  @Rule
+  public TemporaryFolder temporaryFolder = new TemporaryFolder();
+
+  private OzoneConfiguration conf;
+  private SCMStorageConfig scmStorageConfig;
+  private ReconPipelineManager pipelineManager;
+  private ReconContainerManager containerManager;
+
+  @Before
+  public void setUp() throws Exception {
+    conf = new OzoneConfiguration();
+    conf.set(OZONE_METADATA_DIRS,
+        temporaryFolder.newFolder().getAbsolutePath());
+    conf.set(OZONE_SCM_NAMES, "localhost");
+    scmStorageConfig = new ReconStorageConfig(conf);
+    NetworkTopology clusterMap = new NetworkTopologyImpl(conf);
+    EventQueue eventQueue = new EventQueue();
+    NodeManager nodeManager =
+        new SCMNodeManager(conf, scmStorageConfig, eventQueue, clusterMap);
+    pipelineManager = new ReconPipelineManager(conf, nodeManager, eventQueue);
+    containerManager = new ReconContainerManager(conf, pipelineManager);
+  }
+
+  @After
+  public void tearDown() throws IOException {
+    containerManager.close();
+    pipelineManager.close();
+  }
+
+  @Test
+  public void testAddNewContainer() throws IOException {
+    ContainerID containerID = new ContainerID(100L);
+    Pipeline pipeline = getRandomPipeline();
+    pipelineManager.addPipeline(pipeline);
+    ContainerInfo containerInfo =
+        new ContainerInfo.Builder()
+            .setContainerID(containerID.getId())
+            .setNumberOfKeys(10)
+            .setPipelineID(pipeline.getId())
+            .setReplicationFactor(ONE)
+            .setOwner("test")
+            .setState(OPEN)
+            .setReplicationType(STAND_ALONE)
+            .build();
+    ContainerWithPipeline containerWithPipeline =
+        new ContainerWithPipeline(containerInfo, pipeline);
+
+    assertFalse(containerManager.exists(containerID));
+
+    containerManager.addNewContainer(
+        containerID.getId(), containerWithPipeline);
+
+    assertTrue(containerManager.exists(containerID));
+
+    List<ContainerInfo> containers = containerManager.getContainers(OPEN);
+    assertEquals(1, containers.size());
+    assertTrue(containers.get(0).equals(containerInfo));
 
 Review comment:
   ```suggestion
       assertEquals(containerInfo, containers.get(0));
   ```

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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


[GitHub] [hadoop-ozone] adoroszlai commented on issue #503: HDDS-2850. Handle Create container use case in Recon.

Posted by GitBox <gi...@apache.org>.
adoroszlai commented on issue #503: HDDS-2850. Handle Create container use case in Recon.
URL: https://github.com/apache/hadoop-ozone/pull/503#issuecomment-580685993
 
 
   Thanks @avijayanhwx for updating the patch.
   
   @nandakumar131 would you like to review this or can I merge it?

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

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