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

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


The following commit(s) were added to refs/heads/master by this push:
     new 76e3460  HDDS-3222. Add integration test for Recon FSCK. (#850)
76e3460 is described below

commit 76e34604a9fd327074e3084e9c314f5c9a36d704
Author: avijayanhwx <[email protected]>
AuthorDate: Fri Apr 24 10:38:24 2020 -0700

    HDDS-3222. Add integration test for Recon FSCK. (#850)
---
 .../apache/hadoop/ozone/MiniOzoneClusterImpl.java  |   6 +-
 .../hadoop/ozone/recon/TestReconAsPassiveScm.java  |   8 +-
 .../apache/hadoop/ozone/recon/TestReconTasks.java  | 125 +++++++++++++++++++++
 .../ozone/recon/fsck/MissingContainerTask.java     |  24 ++--
 .../hadoop/ozone/recon/scm/PipelineSyncTask.java   |  21 ++--
 .../ozone/recon/scm/ReconContainerManager.java     |   4 +
 .../hadoop/ozone/recon/scm/ReconScmTask.java       |   8 +-
 .../scm/ReconStorageContainerManagerFacade.java    |  14 ++-
 .../hadoop/ozone/recon/tasks/ReconTaskConfig.java  |  69 ++++++++++++
 .../ozone/recon/fsck/TestMissingContainerTask.java |   8 +-
 .../recon/persistence/AbstractReconSqlDBTest.java  |  19 ----
 .../recon/persistence/TestSqlSchemaSetup.java      |  53 +++++++++
 12 files changed, 298 insertions(+), 61 deletions(-)

diff --git 
a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/MiniOzoneClusterImpl.java
 
b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/MiniOzoneClusterImpl.java
index c3d1663..e444355 100644
--- 
a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/MiniOzoneClusterImpl.java
+++ 
b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/MiniOzoneClusterImpl.java
@@ -707,17 +707,17 @@ public class MiniOzoneClusterImpl implements 
MiniOzoneCluster {
         }
         String listOfDirs = String.join(",", dataDirs);
         Path ratisDir = Paths.get(datanodeBaseDir, "data", "ratis");
-        Path wrokDir = Paths.get(datanodeBaseDir, "data", "replication",
+        Path workDir = Paths.get(datanodeBaseDir, "data", "replication",
             "work");
         Files.createDirectories(metaDir);
         Files.createDirectories(ratisDir);
-        Files.createDirectories(wrokDir);
+        Files.createDirectories(workDir);
         dnConf.set(HddsConfigKeys.OZONE_METADATA_DIRS, metaDir.toString());
         dnConf.set(DFSConfigKeysLegacy.DFS_DATANODE_DATA_DIR_KEY, listOfDirs);
         dnConf.set(OzoneConfigKeys.DFS_CONTAINER_RATIS_DATANODE_STORAGE_DIR,
             ratisDir.toString());
         dnConf.set(OzoneConfigKeys.OZONE_CONTAINER_COPY_WORKDIR,
-            wrokDir.toString());
+            workDir.toString());
         if (reconServer != null) {
           OzoneStorageContainerManager reconScm =
               reconServer.getReconStorageContainerManager();
diff --git 
a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconAsPassiveScm.java
 
b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconAsPassiveScm.java
index eabf667..07fbefd 100644
--- 
a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconAsPassiveScm.java
+++ 
b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconAsPassiveScm.java
@@ -81,7 +81,7 @@ public class TestReconAsPassiveScm {
     }
   }
 
-  @Test(timeout = 120000)
+  @Test
   public void testDatanodeRegistrationAndReports() throws Exception {
     ReconStorageContainerManagerFacade reconScm =
         (ReconStorageContainerManagerFacade)
@@ -91,7 +91,7 @@ public class TestReconAsPassiveScm {
     PipelineManager scmPipelineManager = scm.getPipelineManager();
 
     LambdaTestUtils.await(60000, 5000,
-        () -> (reconPipelineManager.getPipelines().size() == 4));
+        () -> (reconPipelineManager.getPipelines().size() >= 4));
 
     // Verify if Recon has all the pipelines from SCM.
     scmPipelineManager.getPipelines().forEach(p -> {
@@ -139,7 +139,7 @@ public class TestReconAsPassiveScm {
         1000, 20000);
   }
 
-  @Test(timeout = 120000)
+  @Test
   public void testReconRestart() throws Exception {
     final OzoneStorageContainerManager reconScm =
             cluster.getReconServer().getReconStorageContainerManager();
@@ -193,7 +193,7 @@ public class TestReconAsPassiveScm {
     assertFalse(
         reconPipelineManager.containsPipeline(pipelineToClose.get().getId()));
 
-    LambdaTestUtils.await(60000, 5000,
+    LambdaTestUtils.await(90000, 5000,
         () -> (newReconScm.getContainerManager()
             .exists(ContainerID.valueof(containerID))));
   }
diff --git 
a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconTasks.java
 
b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconTasks.java
new file mode 100644
index 0000000..08eb960
--- /dev/null
+++ 
b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/recon/TestReconTasks.java
@@ -0,0 +1,125 @@
+/**
+ * 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;
+
+import static 
org.apache.hadoop.hdds.HddsConfigKeys.HDDS_CONTAINER_REPORT_INTERVAL;
+import static 
org.apache.hadoop.hdds.HddsConfigKeys.HDDS_PIPELINE_REPORT_INTERVAL;
+import static 
org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor.ONE;
+import static 
org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType.RATIS;
+import static 
org.apache.hadoop.ozone.container.ozoneimpl.TestOzoneContainer.runTestOzoneContainerViaDataNode;
+import static org.junit.Assert.assertEquals;
+
+import java.util.List;
+
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.scm.XceiverClientGrpc;
+import org.apache.hadoop.hdds.scm.container.ContainerInfo;
+import org.apache.hadoop.hdds.scm.container.ContainerManager;
+import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
+import org.apache.hadoop.hdds.scm.pipeline.PipelineManager;
+import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
+import org.apache.hadoop.ozone.MiniOzoneCluster;
+import org.apache.hadoop.ozone.recon.scm.ReconContainerManager;
+import org.apache.hadoop.ozone.recon.scm.ReconStorageContainerManagerFacade;
+import org.apache.hadoop.test.LambdaTestUtils;
+import org.hadoop.ozone.recon.schema.tables.pojos.MissingContainers;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+
+/**
+ * Integration Tests for Recon's tasks.
+ */
+public class TestReconTasks {
+
+  private MiniOzoneCluster cluster = null;
+  private OzoneConfiguration conf;
+
+  @Rule
+  public TemporaryFolder temporaryFolder = new TemporaryFolder();
+
+  @Before
+  public void init() throws Exception {
+    conf = new OzoneConfiguration();
+    conf.set(HDDS_CONTAINER_REPORT_INTERVAL, "5s");
+    conf.set(HDDS_PIPELINE_REPORT_INTERVAL, "5s");
+    conf.set("ozone.recon.task.missingcontainer.interval", "15s");
+    conf.set("ozone.scm.stale.node.interval", "10s");
+    conf.set("ozone.scm.dead.node.interval", "20s");
+    cluster =  MiniOzoneCluster.newBuilder(conf).setNumDatanodes(1)
+        .includeRecon(true).build();
+    cluster.waitForClusterToBeReady();
+  }
+
+  @After
+  public void shutdown() {
+    if (cluster != null) {
+      cluster.shutdown();
+    }
+  }
+
+  @Test
+  public void testMissingContainerDownNode() throws Exception {
+    ReconStorageContainerManagerFacade reconScm =
+        (ReconStorageContainerManagerFacade)
+            cluster.getReconServer().getReconStorageContainerManager();
+    StorageContainerManager scm = cluster.getStorageContainerManager();
+    PipelineManager reconPipelineManager = reconScm.getPipelineManager();
+    PipelineManager scmPipelineManager = scm.getPipelineManager();
+
+    // Make sure Recon's pipeline state is initialized.
+    LambdaTestUtils.await(60000, 5000,
+        () -> (reconPipelineManager.getPipelines().size() >= 1));
+
+    ContainerManager scmContainerManager = scm.getContainerManager();
+    ReconContainerManager reconContainerManager =
+        (ReconContainerManager) reconScm.getContainerManager();
+    ContainerInfo containerInfo =
+        scmContainerManager.allocateContainer(RATIS, ONE, "test");
+    long containerID = containerInfo.getContainerID();
+    Pipeline pipeline =
+        scmPipelineManager.getPipeline(containerInfo.getPipelineID());
+    XceiverClientGrpc client = new XceiverClientGrpc(pipeline, conf);
+    runTestOzoneContainerViaDataNode(containerID, client);
+
+    // Make sure Recon got the container report with new container.
+    assertEquals(scmContainerManager.getContainerIDs(),
+        reconContainerManager.getContainerIDs());
+
+    // Bring down the Datanode that had the container replica.
+    cluster.shutdownHddsDatanode(pipeline.getFirstNode());
+
+    LambdaTestUtils.await(120000, 10000, () -> {
+      List<MissingContainers> allMissingContainers =
+          reconContainerManager.getContainerSchemaManager()
+              .getAllMissingContainers();
+      return (allMissingContainers.size() == 1);
+    });
+
+    // Restart the Datanode to make sure we remove the missing container.
+    cluster.restartHddsDatanode(pipeline.getFirstNode(), true);
+    LambdaTestUtils.await(120000, 10000, () -> {
+      List<MissingContainers> allMissingContainers =
+          reconContainerManager.getContainerSchemaManager()
+              .getAllMissingContainers();
+      return (allMissingContainers.isEmpty());
+    });
+  }
+}
diff --git 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/fsck/MissingContainerTask.java
 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/fsck/MissingContainerTask.java
index ca4da17..9c1a250 100644
--- 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/fsck/MissingContainerTask.java
+++ 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/fsck/MissingContainerTask.java
@@ -19,17 +19,17 @@
 package org.apache.hadoop.ozone.recon.fsck;
 
 import java.util.Set;
+import java.util.concurrent.TimeUnit;
 
-import javax.inject.Inject;
 
 import 
org.apache.hadoop.hdds.protocol.proto.StorageContainerDatanodeProtocolProtos.ContainerReplicaProto.State;
 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.ContainerReplica;
-import org.apache.hadoop.hdds.scm.server.OzoneStorageContainerManager;
 import org.apache.hadoop.ozone.recon.persistence.ContainerSchemaManager;
 import org.apache.hadoop.ozone.recon.scm.ReconScmTask;
+import org.apache.hadoop.ozone.recon.tasks.ReconTaskConfig;
 import org.apache.hadoop.util.Time;
 import org.hadoop.ozone.recon.schema.tables.daos.ReconTaskStatusDao;
 import org.slf4j.Logger;
@@ -47,16 +47,18 @@ public class MissingContainerTask extends ReconScmTask {
 
   private ContainerManager containerManager;
   private ContainerSchemaManager containerSchemaManager;
-  private static final long INTERVAL = 5 * 60 * 1000L;
+  private final long interval;
 
-  @Inject
   public MissingContainerTask(
-      OzoneStorageContainerManager ozoneStorageContainerManager,
+      ContainerManager containerManager,
       ReconTaskStatusDao reconTaskStatusDao,
-      ContainerSchemaManager containerSchemaManager) {
+      ContainerSchemaManager containerSchemaManager,
+      ReconTaskConfig reconTaskConfig) {
     super(reconTaskStatusDao);
     this.containerSchemaManager = containerSchemaManager;
-    this.containerManager = ozoneStorageContainerManager.getContainerManager();
+    this.containerManager = containerManager;
+    this.interval = TimeUnit.SECONDS.toMillis(
+        reconTaskConfig.getMissingContainerTaskInterval());
   }
 
   public synchronized void run() {
@@ -72,7 +74,7 @@ public class MissingContainerTask extends ReconScmTask {
         LOG.info("Missing Container task Thread took {} milliseconds for" +
                 " processing {} containers.", Time.monotonicNow() - start,
             containerIds.size());
-        wait(INTERVAL);
+        wait(interval);
       }
     } catch (Throwable t) {
       LOG.error("Exception in Missing Container task Thread.", t);
@@ -92,13 +94,15 @@ public class MissingContainerTask extends ReconScmTask {
           containerSchemaManager.isMissingContainer(containerID.getId());
       if (CollectionUtils.isEmpty(containerReplicas) || isAllUnhealthy) {
         if (!isMissingContainer) {
-          LOG.info("Found a missing container with ID {}. Adding it to the " +
-              "database", containerID.getId());
+          LOG.info("Found a missing container with ID {}.",
+              containerID.getId());
           containerSchemaManager.addMissingContainer(containerID.getId(),
               currentTime);
         }
       } else {
         if (isMissingContainer) {
+          LOG.info("Missing container with ID {} is no longer missing.",
+              containerID.getId());
           containerSchemaManager.deleteMissingContainer(containerID.getId());
         }
       }
diff --git 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/PipelineSyncTask.java
 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/PipelineSyncTask.java
index 80de0ae..ecd1b45 100644
--- 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/PipelineSyncTask.java
+++ 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/PipelineSyncTask.java
@@ -19,12 +19,11 @@
 package org.apache.hadoop.ozone.recon.scm;
 
 import java.util.List;
-
-import javax.inject.Inject;
+import java.util.concurrent.TimeUnit;
 
 import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
-import org.apache.hadoop.hdds.scm.server.OzoneStorageContainerManager;
 import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
+import org.apache.hadoop.ozone.recon.tasks.ReconTaskConfig;
 import org.apache.hadoop.util.Time;
 import org.hadoop.ozone.recon.schema.tables.daos.ReconTaskStatusDao;
 import org.slf4j.Logger;
@@ -41,17 +40,17 @@ public class PipelineSyncTask extends ReconScmTask {
 
   private StorageContainerServiceProvider scmClient;
   private ReconPipelineManager reconPipelineManager;
-  private static final long INTERVAL = 10 * 60 * 1000L;
+  private final long interval;
 
-  @Inject
-  public PipelineSyncTask(
-      OzoneStorageContainerManager storageContainerManager,
+  public PipelineSyncTask(ReconPipelineManager pipelineManager,
       StorageContainerServiceProvider scmClient,
-      ReconTaskStatusDao reconTaskStatusDao) {
+      ReconTaskStatusDao reconTaskStatusDao,
+      ReconTaskConfig reconTaskConfig) {
     super(reconTaskStatusDao);
     this.scmClient = scmClient;
-    this.reconPipelineManager = (ReconPipelineManager)
-        storageContainerManager.getPipelineManager();
+    this.reconPipelineManager = pipelineManager;
+    this.interval = TimeUnit.SECONDS.toMillis(
+        reconTaskConfig.getPipelineSyncTaskInterval());
   }
 
   @Override
@@ -64,7 +63,7 @@ public class PipelineSyncTask extends ReconScmTask {
         LOG.info("Pipeline sync Thread took {} milliseconds.",
             Time.monotonicNow() - start);
         recordSingleRunCompletion();
-        wait(INTERVAL);
+        wait(interval);
       }
     } catch (Throwable t) {
       LOG.error("Exception in Pipeline sync Thread.", t);
diff --git 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconContainerManager.java
 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconContainerManager.java
index aa524b0..72d1548 100644
--- 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconContainerManager.java
+++ 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconContainerManager.java
@@ -147,4 +147,8 @@ public class ReconContainerManager extends 
SCMContainerManager {
     containerSchemaManager.upsertContainerHistory(containerID.getId(),
         datanodeHost, currentTime);
   }
+
+  public ContainerSchemaManager getContainerSchemaManager() {
+    return containerSchemaManager;
+  }
 }
diff --git 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconScmTask.java
 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconScmTask.java
index 27d9892..df21c21 100644
--- 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconScmTask.java
+++ 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconScmTask.java
@@ -23,8 +23,6 @@ import 
org.hadoop.ozone.recon.schema.tables.pojos.ReconTaskStatus;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
-import com.google.inject.Inject;
-
 /**
  * Any background task that keeps SCM's metadata up to date.
  */
@@ -35,12 +33,11 @@ public abstract class ReconScmTask {
   private ReconTaskStatusDao reconTaskStatusDao;
   private volatile boolean running;
 
-  @Inject
-  public ReconScmTask(ReconTaskStatusDao reconTaskStatusDao) {
+  protected ReconScmTask(ReconTaskStatusDao reconTaskStatusDao) {
     this.reconTaskStatusDao = reconTaskStatusDao;
   }
 
-  public void register() {
+  private void register() {
     String taskName = getTaskName();
     if (!reconTaskStatusDao.existsById(taskName)) {
       ReconTaskStatus reconTaskStatusRecord = new ReconTaskStatus(
@@ -54,6 +51,7 @@ public abstract class ReconScmTask {
    * Start underlying start thread.
    */
   public synchronized void start() {
+    register();
     if (!isRunning()) {
       LOG.info("Starting {} Thread.", getTaskName());
       running = true;
diff --git 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconStorageContainerManagerFacade.java
 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconStorageContainerManagerFacade.java
index 7800abb..d6af3a8 100644
--- 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconStorageContainerManagerFacade.java
+++ 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/scm/ReconStorageContainerManagerFacade.java
@@ -53,7 +53,7 @@ import org.apache.hadoop.io.IOUtils;
 import org.apache.hadoop.ozone.recon.fsck.MissingContainerTask;
 import org.apache.hadoop.ozone.recon.persistence.ContainerSchemaManager;
 import org.apache.hadoop.ozone.recon.spi.StorageContainerServiceProvider;
-
+import org.apache.hadoop.ozone.recon.tasks.ReconTaskConfig;
 import com.google.inject.Inject;
 import static 
org.apache.hadoop.hdds.recon.ReconConfigKeys.RECON_SCM_CONFIG_PREFIX;
 import static 
org.apache.hadoop.hdds.scm.server.StorageContainerManager.buildRpcServerStartMessage;
@@ -154,15 +154,17 @@ public class ReconStorageContainerManagerFacade
     eventQueue.addHandler(SCMEvents.CLOSE_CONTAINER, closeContainerHandler);
     eventQueue.addHandler(SCMEvents.NEW_NODE, newNodeHandler);
 
+    ReconTaskConfig reconTaskConfig = conf.getObject(ReconTaskConfig.class);
     reconScmTasks.add(new PipelineSyncTask(
-        this,
+        pipelineManager,
         scmServiceProvider,
-        reconTaskStatusDao));
+        reconTaskStatusDao,
+        reconTaskConfig));
     reconScmTasks.add(new MissingContainerTask(
-        this,
+        containerManager,
         reconTaskStatusDao,
-        containerSchemaManager));
-    reconScmTasks.forEach(ReconScmTask::register);
+        containerSchemaManager,
+        reconTaskConfig));
   }
 
   /**
diff --git 
a/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/tasks/ReconTaskConfig.java
 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/tasks/ReconTaskConfig.java
new file mode 100644
index 0000000..688e3ac
--- /dev/null
+++ 
b/hadoop-ozone/recon/src/main/java/org/apache/hadoop/ozone/recon/tasks/ReconTaskConfig.java
@@ -0,0 +1,69 @@
+/*
+ * 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.tasks;
+
+import java.util.concurrent.TimeUnit;
+
+import org.apache.hadoop.hdds.conf.Config;
+import org.apache.hadoop.hdds.conf.ConfigGroup;
+import org.apache.hadoop.hdds.conf.ConfigTag;
+import org.apache.hadoop.hdds.conf.ConfigType;
+
+/**
+ * The configuration class for the Recon tasks.
+ */
+@ConfigGroup(prefix = "ozone.recon.task")
+public class ReconTaskConfig {
+
+  @Config(key = "pipelinesync.interval",
+      type = ConfigType.TIME, timeUnit = TimeUnit.SECONDS,
+      defaultValue = "600s",
+      tags = { ConfigTag.RECON, ConfigTag.OZONE },
+      description = "The time interval of periodic sync of pipeline state " +
+          "from SCM to Recon."
+  )
+  private long pipelineSyncTaskInterval;
+
+  public long getPipelineSyncTaskInterval() {
+    return pipelineSyncTaskInterval;
+  }
+
+  public void setPipelineSyncTaskInterval(long pipelineSyncTaskInterval) {
+    this.pipelineSyncTaskInterval = pipelineSyncTaskInterval;
+  }
+
+  @Config(key = "missingcontainer.interval",
+      type = ConfigType.TIME, timeUnit = TimeUnit.SECONDS,
+      defaultValue = "300s",
+      tags = { ConfigTag.RECON, ConfigTag.OZONE },
+      description = "The time interval of the periodic check for " +
+          "containers with zero replicas in the cluster as reported by " +
+          "Datanodes."
+  )
+  private long missingContainerTaskInterval;
+
+  public long getMissingContainerTaskInterval() {
+    return missingContainerTaskInterval;
+  }
+
+  public void setMissingContainerTaskInterval(long interval) {
+    this.missingContainerTaskInterval = interval;
+  }
+
+}
diff --git 
a/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/fsck/TestMissingContainerTask.java
 
b/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/fsck/TestMissingContainerTask.java
index aea68a7..153f05a 100644
--- 
a/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/fsck/TestMissingContainerTask.java
+++ 
b/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/fsck/TestMissingContainerTask.java
@@ -33,6 +33,7 @@ import org.apache.hadoop.hdds.scm.container.ContainerManager;
 import org.apache.hadoop.hdds.scm.container.ContainerReplica;
 import org.apache.hadoop.ozone.recon.persistence.ContainerSchemaManager;
 import org.apache.hadoop.ozone.recon.scm.ReconStorageContainerManagerFacade;
+import org.apache.hadoop.ozone.recon.tasks.ReconTaskConfig;
 import org.apache.hadoop.test.LambdaTestUtils;
 import org.hadoop.ozone.recon.schema.ContainerSchemaDefinition;
 import org.hadoop.ozone.recon.schema.tables.daos.ContainerHistoryDao;
@@ -87,10 +88,11 @@ public class TestMissingContainerTask extends 
AbstractReconSqlDBTest {
 
     long currentTime = System.currentTimeMillis();
     ReconTaskStatusDao reconTaskStatusDao = getDao(ReconTaskStatusDao.class);
+    ReconTaskConfig reconTaskConfig = new ReconTaskConfig();
+    reconTaskConfig.setMissingContainerTaskInterval(60);
     MissingContainerTask missingContainerTask =
-        new MissingContainerTask(scmMock, reconTaskStatusDao,
-            containerSchemaManager);
-    missingContainerTask.register();
+        new MissingContainerTask(scmMock.getContainerManager(),
+            reconTaskStatusDao, containerSchemaManager, reconTaskConfig);
     missingContainerTask.start();
 
     LambdaTestUtils.await(6000, 1000, () ->
diff --git 
a/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/persistence/AbstractReconSqlDBTest.java
 
b/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/persistence/AbstractReconSqlDBTest.java
index ec6610c..f8768dc 100644
--- 
a/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/persistence/AbstractReconSqlDBTest.java
+++ 
b/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/persistence/AbstractReconSqlDBTest.java
@@ -17,9 +17,6 @@
  */
 package org.apache.hadoop.ozone.recon.persistence;
 
-import static 
org.apache.hadoop.ozone.recon.ReconControllerModule.ReconDaoBindingModule.RECON_DAO_LIST;
-import static org.junit.Assert.assertNotNull;
-
 import java.io.File;
 import java.io.IOException;
 import java.sql.Connection;
@@ -40,7 +37,6 @@ import org.jooq.impl.DefaultConfiguration;
 import org.junit.Assert;
 import org.junit.Before;
 import org.junit.Rule;
-import org.junit.Test;
 import org.junit.rules.TemporaryFolder;
 
 import com.google.inject.AbstractModule;
@@ -147,21 +143,6 @@ public class AbstractReconSqlDBTest {
   }
 
   /**
-   * Make sure schema was created correctly.
-   * @throws SQLException
-   */
-  @Test
-  public void testSchemaSetup() throws SQLException {
-    assertNotNull(injector);
-    assertNotNull(getConfiguration());
-    assertNotNull(dslContext);
-    assertNotNull(getConnection());
-    RECON_DAO_LIST.forEach(dao -> {
-      assertNotNull(getDao(dao));
-    });
-  }
-
-  /**
    * Local Sqlite datasource provider.
    */
   public static class DataSourceConfigurationProvider implements
diff --git 
a/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/persistence/TestSqlSchemaSetup.java
 
b/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/persistence/TestSqlSchemaSetup.java
new file mode 100644
index 0000000..19f8d70
--- /dev/null
+++ 
b/hadoop-ozone/recon/src/test/java/org/apache/hadoop/ozone/recon/persistence/TestSqlSchemaSetup.java
@@ -0,0 +1,53 @@
+/*
+ * 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.persistence;
+
+import static 
org.apache.hadoop.ozone.recon.ReconControllerModule.ReconDaoBindingModule.RECON_DAO_LIST;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertNotNull;
+
+import java.sql.SQLException;
+
+import org.hadoop.ozone.recon.schema.tables.daos.ReconTaskStatusDao;
+import org.hadoop.ozone.recon.schema.tables.pojos.ReconTaskStatus;
+import org.junit.Test;
+
+/**
+ * Class to test basic SQL schema setup.
+ */
+public class TestSqlSchemaSetup extends AbstractReconSqlDBTest {
+
+  /**
+   * Make sure schema was created correctly.
+   * @throws SQLException
+   */
+  @Test
+  public void testSchemaSetup() throws SQLException {
+    assertNotNull(getInjector());
+    assertNotNull(getConfiguration());
+    assertNotNull(getDslContext());
+    assertNotNull(getConnection());
+    RECON_DAO_LIST.forEach(dao -> {
+      assertNotNull(getDao(dao));
+    });
+    ReconTaskStatusDao dao = getDao(ReconTaskStatusDao.class);
+    dao.insert(new ReconTaskStatus("TestTask", 1L, 2L));
+    assertEquals(1, dao.findAll().size());
+  }
+}


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to