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 2022/10/25 07:26:05 UTC

[GitHub] [ozone] myskov commented on a diff in pull request #3874: HDDS-7383. Basic framework of DiskBalancerService

myskov commented on code in PR #3874:
URL: https://github.com/apache/ozone/pull/3874#discussion_r1004088332


##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/volume/StorageVolume.java:
##########
@@ -62,6 +62,8 @@
   private static final Logger LOG =
       LoggerFactory.getLogger(StorageVolume.class);
 
+  public static final String TMP_DIR = "tmp";

Review Comment:
   Probably it makes sense to parametrize this (like hdds.datanode.replication.work.dir for example)



##########
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/diskbalancer/DiskBalancerYaml.java:
##########
@@ -0,0 +1,152 @@
+/*
+ * 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.container.diskbalancer;
+
+import org.yaml.snakeyaml.DumperOptions;
+import org.yaml.snakeyaml.Yaml;
+
+import java.io.File;
+import java.io.FileInputStream;
+import java.io.FileOutputStream;
+import java.io.IOException;
+import java.io.OutputStreamWriter;
+import java.io.Writer;
+import java.nio.charset.StandardCharsets;
+
+/**
+ * Class for creating diskbalancer.info file in yaml format.
+ */
+
+public final class DiskBalancerYaml {
+
+  private DiskBalancerYaml() {
+    // static helper methods only, no state.
+  }
+
+  /**
+   * Creates a yaml file to store DiskBalancer info.
+   *
+   * @param diskBalancerInfo {@link DiskBalancerInfo}
+   * @param path            Path to diskBalancer.info file
+   */
+  public static void createDiskBalancerInfoFile(
+      DiskBalancerInfo diskBalancerInfo, File path)
+      throws IOException {
+    DumperOptions options = new DumperOptions();
+    options.setPrettyFlow(true);
+    options.setDefaultFlowStyle(DumperOptions.FlowStyle.FLOW);
+    Yaml yaml = new Yaml(options);
+
+    try (Writer writer = new OutputStreamWriter(
+        new FileOutputStream(path), StandardCharsets.UTF_8)) {
+      yaml.dump(getDiskBalancerInfoYaml(diskBalancerInfo), writer);
+    }
+  }
+
+  /**
+   * Read DiskBalancerConfiguration from file.
+   */
+  public static DiskBalancerInfo readDiskBalancerInfoFile(File path)
+      throws IOException {
+    DiskBalancerInfo diskBalancerInfo;
+
+    try (FileInputStream inputFileStream = new FileInputStream(path)) {
+      Yaml yaml = new Yaml();
+      DiskBalancerInfoYaml diskBalancerInfoYaml;
+      try {
+        diskBalancerInfoYaml =
+            yaml.loadAs(inputFileStream, DiskBalancerInfoYaml.class);
+      } catch (Exception e) {
+        throw new IOException("Unable to parse yaml file.", e);
+      }
+
+      diskBalancerInfo = new DiskBalancerInfo(
+          diskBalancerInfoYaml.isShouldRun(),
+          diskBalancerInfoYaml.getThreshold(),
+          diskBalancerInfoYaml.getBandwidthInMB(),
+          diskBalancerInfoYaml.getParallelThread());
+    }
+
+    return diskBalancerInfo;
+  }
+
+  /**
+   * Datanode DiskBalancer Info to be written to the yaml file.
+   */
+  public static class DiskBalancerInfoYaml {
+    private boolean shouldRun;
+    private double threshold;
+    private long bandwidthInMB;
+    private int parallelThread;
+
+    public DiskBalancerInfoYaml() {
+      // Needed for snake-yaml introspection.
+    }
+
+    @SuppressWarnings({"parameternumber", "java:S107"}) // required for yaml

Review Comment:
   is suppressing parameternumber really needed? There are only 4 params but the threshold is 6 or 7 AFAIR



##########
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/storage/DiskBalancerConfiguration.java:
##########
@@ -36,6 +38,17 @@ public final class DiskBalancerConfiguration {
   private static final Logger LOG =
       LoggerFactory.getLogger(DiskBalancerConfiguration.class);
 
+  // The path where datanode diskBalancer's conf is to be written to.
+  public static final String HDDS_DATANODE_DISK_BALANCER_INFO_DIR =

Review Comment:
   These configuration properties should be put to ozone-default.xml



##########
hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/server/SCMClientProtocolServer.java:
##########
@@ -1159,15 +1159,29 @@ public List<HddsProtos.DatanodeDiskBalancerInfoProto> getDiskBalancerStatus(
   public List<DatanodeAdminError> startDiskBalancer(Optional<Double> threshold,
       Optional<Long> bandwidthInMB, Optional<Integer> parallelThread,
       Optional<List<String>> hosts) throws IOException {
-    // TODO: Send message to datanodes
-    return new ArrayList<DatanodeAdminError>();
+    // check admin authorisation
+    try {
+      getScm().checkAdminAccess(getRemoteUser());
+    } catch (IOException e) {
+      LOG.error("Authorization failed", e);
+      throw e;
+    }
+
+    return scm.getDiskBalancerManager()
+        .startDiskBalancer(threshold, bandwidthInMB, parallelThread, hosts);
   }
 
   @Override
   public List<DatanodeAdminError> stopDiskBalancer(Optional<List<String>> hosts)
       throws IOException {
-    // TODO: Send message to datanodes
-    return new ArrayList<DatanodeAdminError>();
+    // check admin authorisation

Review Comment:
   I believe this comment is redundant



##########
hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/server/SCMClientProtocolServer.java:
##########
@@ -1159,15 +1159,29 @@ public List<HddsProtos.DatanodeDiskBalancerInfoProto> getDiskBalancerStatus(
   public List<DatanodeAdminError> startDiskBalancer(Optional<Double> threshold,
       Optional<Long> bandwidthInMB, Optional<Integer> parallelThread,
       Optional<List<String>> hosts) throws IOException {
-    // TODO: Send message to datanodes
-    return new ArrayList<DatanodeAdminError>();
+    // check admin authorisation

Review Comment:
   I believe this comment is redundant 



##########
hadoop-hdds/container-service/src/test/java/org/apache/hadoop/ozone/container/diskbalancer/DiskBalancerServiceTestImpl.java:
##########
@@ -0,0 +1,107 @@
+/*
+ * 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.container.diskbalancer;
+
+import java.io.IOException;
+import java.util.concurrent.CountDownLatch;
+import java.util.concurrent.Future;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicInteger;
+
+import org.apache.hadoop.hdds.conf.ConfigurationSource;
+import org.apache.hadoop.ozone.container.ozoneimpl.OzoneContainer;
+
+import com.google.common.annotations.VisibleForTesting;
+import com.google.common.util.concurrent.ThreadFactoryBuilder;
+
+/**
+ * A test class implementation for {@link DiskBalancerService}.
+ */
+public class DiskBalancerServiceTestImpl extends DiskBalancerService {
+
+  // the service timeout
+  private static final int SERVICE_TIMEOUT_IN_MILLISECONDS = 0;
+
+  // tests only
+  private CountDownLatch latch;
+  private Thread testingThread;
+  private AtomicInteger numOfProcessed = new AtomicInteger(0);
+
+  public DiskBalancerServiceTestImpl(OzoneContainer container,
+      int serviceInterval, ConfigurationSource conf, int threadCount)
+      throws IOException {
+    super(container, serviceInterval, SERVICE_TIMEOUT_IN_MILLISECONDS,
+        TimeUnit.MILLISECONDS, threadCount, conf);
+  }
+
+  @VisibleForTesting
+  public void runBalanceTasks() {
+    if (latch.getCount() > 0) {
+      this.latch.countDown();
+    } else {
+      throw new IllegalStateException("Count already reaches zero");
+    }
+  }
+
+  @VisibleForTesting

Review Comment:
   do we need VisibleForTesting annotation in test code?



##########
hadoop-hdds/container-service/src/test/java/org/apache/hadoop/ozone/container/diskbalancer/DiskBalancerServiceTestImpl.java:
##########
@@ -0,0 +1,107 @@
+/*
+ * 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.container.diskbalancer;
+
+import java.io.IOException;
+import java.util.concurrent.CountDownLatch;
+import java.util.concurrent.Future;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicInteger;
+
+import org.apache.hadoop.hdds.conf.ConfigurationSource;
+import org.apache.hadoop.ozone.container.ozoneimpl.OzoneContainer;
+
+import com.google.common.annotations.VisibleForTesting;
+import com.google.common.util.concurrent.ThreadFactoryBuilder;
+
+/**
+ * A test class implementation for {@link DiskBalancerService}.
+ */
+public class DiskBalancerServiceTestImpl extends DiskBalancerService {
+
+  // the service timeout
+  private static final int SERVICE_TIMEOUT_IN_MILLISECONDS = 0;
+
+  // tests only
+  private CountDownLatch latch;
+  private Thread testingThread;
+  private AtomicInteger numOfProcessed = new AtomicInteger(0);
+
+  public DiskBalancerServiceTestImpl(OzoneContainer container,
+      int serviceInterval, ConfigurationSource conf, int threadCount)
+      throws IOException {
+    super(container, serviceInterval, SERVICE_TIMEOUT_IN_MILLISECONDS,
+        TimeUnit.MILLISECONDS, threadCount, conf);
+  }
+
+  @VisibleForTesting

Review Comment:
   do we need VisibleForTesting annotation in test code?



##########
hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/server/SCMClientProtocolServer.java:
##########
@@ -1176,8 +1190,16 @@ public List<DatanodeAdminError> updateDiskBalancerConfiguration(
       Optional<Double> threshold, Optional<Long> bandwidthInMB,
       Optional<Integer> parallelThread, Optional<List<String>> hosts)
       throws IOException {
-    // TODO: Send message to datanodes
-    return new ArrayList<DatanodeAdminError>();
+    // check admin authorisation

Review Comment:
   I believe this comment is redundant



-- 
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.

To unsubscribe, e-mail: issues-unsubscribe@ozone.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


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