Kaynağa Gözat

HDDS-976: Parse network topology from yaml file. Contributed by Junjie Chen. (#661)

Signed-off-by: Xiaoyu Yao <xyao@apache.org>
Chen, Junjie 6 yıl önce
ebeveyn
işleme
518f47bf9b

+ 5 - 0
hadoop-hdds/common/pom.xml

@@ -137,6 +137,11 @@ http://maven.apache.org/xsd/maven-4.0.0.xsd">
       <artifactId>opentracing-util</artifactId>
       <version>0.31.0</version>
     </dependency>
+    <dependency>
+      <groupId>org.yaml</groupId>
+      <artifactId>snakeyaml</artifactId>
+      <version>1.16</version>
+    </dependency>
   </dependencies>
 
   <build>

+ 2 - 0
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/ScmConfigKeys.java

@@ -368,6 +368,8 @@ public final class ScmConfigKeys {
       "hdds.scm.http.kerberos.keytab";
 
   // Network topology
+  public static final String OZONE_SCM_NETWORK_TOPOLOGY_SCHEMA_FILE_TYPE =
+          "ozone.scm.network.topology.schema.file.type";
   public static final String OZONE_SCM_NETWORK_TOPOLOGY_SCHEMA_FILE =
       "ozone.scm.network.topology.schema.file";
   public static final String OZONE_SCM_NETWORK_TOPOLOGY_SCHEMA_FILE_DEFAULT =

+ 39 - 4
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/net/NodeSchema.java

@@ -19,6 +19,8 @@ package org.apache.hadoop.hdds.scm.net;
 
 import org.apache.hadoop.HadoopIllegalArgumentException;
 
+import java.util.List;
+
 /**
  * Network topology schema to housekeeper relevant information.
  */
@@ -59,13 +61,15 @@ public final class NodeSchema {
   }
 
   // default cost
-  private final int cost;
+  private int cost;
   // layer Type, mandatory property
-  private final LayerType type;
+  private LayerType type;
   // default name, can be null or ""
-  private final String defaultName;
+  private String defaultName;
   // layer prefix, can be null or ""
-  private final String prefix;
+  private String prefix;
+  // sublayer
+  private List<NodeSchema> sublayer;
 
   /**
    * Builder for NodeSchema.
@@ -123,6 +127,14 @@ public final class NodeSchema {
     this.defaultName = defaultName;
   }
 
+  /**
+   * Constructor. This constructor is only used when build NodeSchema from
+   * YAML file.
+   */
+  public NodeSchema() {
+    this.type = LayerType.INNER_NODE;
+  }
+
   public boolean matchPrefix(String name) {
     if (name == null || name.isEmpty() || prefix == null || prefix.isEmpty()) {
       return false;
@@ -134,15 +146,38 @@ public final class NodeSchema {
     return this.type;
   }
 
+  public void setType(LayerType type) {
+    this.type = type;
+  }
+
   public String getPrefix() {
     return this.prefix;
   }
 
+  public void setPrefix(String prefix) {
+    this.prefix = prefix;
+  }
+
   public String getDefaultName() {
     return this.defaultName;
   }
 
+  public void setDefaultName(String name) {
+    this.defaultName = name;
+  }
+
   public int getCost() {
     return this.cost;
   }
+  public void setCost(int cost) {
+    this.cost = cost;
+  }
+
+  public void setSublayer(List<NodeSchema> sublayer) {
+    this.sublayer = sublayer;
+  }
+
+  public List<NodeSchema> getSublayer() {
+    return sublayer;
+  }
 }

+ 85 - 1
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/net/NodeSchemaLoader.java

@@ -30,6 +30,7 @@ import javax.xml.parsers.DocumentBuilder;
 import javax.xml.parsers.DocumentBuilderFactory;
 import javax.xml.parsers.ParserConfigurationException;
 import java.io.File;
+import java.io.FileInputStream;
 import java.io.IOException;
 import java.util.ArrayList;
 import java.util.HashMap;
@@ -37,6 +38,7 @@ import java.util.List;
 import java.util.Map;
 
 import  org.apache.hadoop.hdds.scm.net.NodeSchema.LayerType;
+import org.yaml.snakeyaml.Yaml;
 
 /**
  * A Network topology layer schema loading tool that loads user defined network
@@ -95,7 +97,7 @@ public final class NodeSchemaLoader {
    * @param schemaFilePath path of schema file
    * @return all valid node schemas defined in schema file
    */
-  public NodeSchemaLoadResult loadSchemaFromFile(String schemaFilePath)
+  public NodeSchemaLoadResult loadSchemaFromXml(String schemaFilePath)
       throws IllegalArgumentException {
     try {
       File schemaFile = new File(schemaFilePath);
@@ -165,6 +167,88 @@ public final class NodeSchemaLoader {
     return schemaList;
   }
 
+  /**
+   * Load user defined network layer schemas from a YAML configuration file.
+   * @param schemaFilePath path of schema file
+   * @return all valid node schemas defined in schema file
+   */
+  public NodeSchemaLoadResult loadSchemaFromYaml(String schemaFilePath)
+          throws IllegalArgumentException {
+    try {
+      File schemaFile = new File(schemaFilePath);
+      if (!schemaFile.exists()) {
+        String msg = "Network topology layer schema file " + schemaFilePath +
+                " is not found.";
+        LOG.warn(msg);
+        throw new IllegalArgumentException(msg);
+      }
+      return loadSchemaFromYaml(schemaFile);
+    } catch (Exception e) {
+      throw new IllegalArgumentException("Fail to load network topology node"
+              + " schema file: " + schemaFilePath + " , error:"
+              + e.getMessage());
+    }
+  }
+
+  /**
+   * Load network topology layer schemas from a YAML configuration file.
+   * @param schemaFile schema file
+   * @return all valid node schemas defined in schema file
+   * @throws ParserConfigurationException ParserConfigurationException happen
+   * @throws IOException no such schema file
+   * @throws SAXException xml file has some invalid elements
+   * @throws IllegalArgumentException xml file content is logically invalid
+   */
+  private NodeSchemaLoadResult loadSchemaFromYaml(File schemaFile) {
+    LOG.info("Loading network topology layer schema file {}", schemaFile);
+    NodeSchemaLoadResult finalSchema;
+
+    try {
+      Yaml yaml = new Yaml();
+      NodeSchema nodeTree;
+
+      try (FileInputStream fileInputStream = new FileInputStream(schemaFile)) {
+        nodeTree = yaml.loadAs(fileInputStream, NodeSchema.class);
+      }
+      List<NodeSchema> schemaList = new ArrayList<>();
+      if (nodeTree.getType() != LayerType.ROOT) {
+        throw new IllegalArgumentException("First layer is not a ROOT node."
+                + " schema file: " + schemaFile.getAbsolutePath());
+      }
+      schemaList.add(nodeTree);
+      if (nodeTree.getSublayer() != null) {
+        nodeTree = nodeTree.getSublayer().get(0);
+      }
+
+      while (nodeTree != null) {
+        if (nodeTree.getType() == LayerType.LEAF_NODE
+                && nodeTree.getSublayer() != null) {
+          throw new IllegalArgumentException("Leaf node in the middle of path."
+                  + " schema file: " + schemaFile.getAbsolutePath());
+        }
+        if (nodeTree.getType() == LayerType.ROOT) {
+          throw new IllegalArgumentException("Multiple root nodes are defined."
+                  + " schema file: " + schemaFile.getAbsolutePath());
+        }
+        schemaList.add(nodeTree);
+        if (nodeTree.getSublayer() != null) {
+          nodeTree = nodeTree.getSublayer().get(0);
+        } else {
+          break;
+        }
+      }
+      finalSchema = new NodeSchemaLoadResult(schemaList, true);
+    } catch (RuntimeException e) {
+      throw  e;
+    } catch (Exception e) {
+      throw new IllegalArgumentException("Fail to load network topology node"
+              + " schema file: " + schemaFile.getAbsolutePath() + " , error:"
+              + e.getMessage());
+    }
+
+    return finalSchema;
+  }
+
   /**
    * Load layoutVersion from root element in the XML configuration file.
    * @param root root element

+ 8 - 1
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/net/NodeSchemaManager.java

@@ -59,13 +59,20 @@ public final class NodeSchemaManager {
     /**
      * Load schemas from network topology schema configuration file
      */
+    String schemaFileType = conf.get(
+            ScmConfigKeys.OZONE_SCM_NETWORK_TOPOLOGY_SCHEMA_FILE_TYPE);
+
     String schemaFile = conf.get(
         ScmConfigKeys.OZONE_SCM_NETWORK_TOPOLOGY_SCHEMA_FILE,
         ScmConfigKeys.OZONE_SCM_NETWORK_TOPOLOGY_SCHEMA_FILE_DEFAULT);
 
     NodeSchemaLoadResult result;
     try {
-      result = NodeSchemaLoader.getInstance().loadSchemaFromFile(schemaFile);
+      if (schemaFileType.toLowerCase().compareTo("yaml") == 0) {
+        result = NodeSchemaLoader.getInstance().loadSchemaFromYaml(schemaFile);
+      } else {
+        result = NodeSchemaLoader.getInstance().loadSchemaFromXml(schemaFile);
+      }
       allSchema = result.getSchemaList();
       enforcePrefix = result.isEnforePrefix();
       maxLevel = allSchema.size();

+ 61 - 0
hadoop-hdds/common/src/main/resources/network-topology-default.yaml

@@ -0,0 +1,61 @@
+# 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
+#
+#     http://www.apache.org/licenses/LICENSE-2.0
+#
+# 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.
+
+---
+# Cost: The cost of crossing this layer.
+# The value should be positive integer or 0. This field is optional.
+# When it's not defined, it's value is default "1".
+cost: 1
+
+# The prefix of this layer.
+# If the prefix is "dc", then every name in this layer should start with "dc",
+# such as "dc1", "dc2".
+# Note that unlike XML schema, the prefix must be specified explicitly if the type is InnerNode.
+prefix: /
+
+# Layer type, optional field, default value InnerNode.
+# Current value range : {ROOT, INNER_NODE, LEAF_NODE}
+type: ROOT
+
+# Layer name
+defaultName: root
+
+# Sub layer
+# The sub layer property defines as a list which can reflect a node tree, though
+# in schema template it always has only one child.
+sublayer:
+  -
+    cost: 1
+    prefix: dc
+    defaultName: datacenter
+    type: INNER_NODE
+    sublayer:
+      -
+        cost: 1
+        prefix: rack
+        defaultName: rack
+        type: INNER_NODE
+        sublayer:
+            -
+              cost: 1
+              prefix: ng
+              defaultName: nodegroup
+              type: INNER_NODE
+              sublayer:
+                -
+                  defaultName: node
+                  type: LEAF_NODE
+                  prefix: node
+...

+ 3 - 3
hadoop-hdds/common/src/test/java/org/apache/hadoop/hdds/scm/net/TestNodeSchemaLoader.java

@@ -44,7 +44,7 @@ public class TestNodeSchemaLoader {
     try {
       String filePath = classLoader.getResource(
           "./networkTopologyTestFiles/" + schemaFile).getPath();
-      NodeSchemaLoader.getInstance().loadSchemaFromFile(filePath);
+      NodeSchemaLoader.getInstance().loadSchemaFromXml(filePath);
       fail("expect exceptions");
     } catch (Throwable e) {
       assertTrue(e.getMessage().contains(errMsg));
@@ -83,7 +83,7 @@ public class TestNodeSchemaLoader {
     try {
       String filePath = classLoader.getResource(
           "./networkTopologyTestFiles/good.xml").getPath();
-      NodeSchemaLoader.getInstance().loadSchemaFromFile(filePath);
+      NodeSchemaLoader.getInstance().loadSchemaFromXml(filePath);
     } catch (Throwable e) {
       fail("should succeed");
     }
@@ -94,7 +94,7 @@ public class TestNodeSchemaLoader {
     String filePath = classLoader.getResource(
         "./networkTopologyTestFiles/good.xml").getPath() + ".backup";
     try {
-      NodeSchemaLoader.getInstance().loadSchemaFromFile(filePath);
+      NodeSchemaLoader.getInstance().loadSchemaFromXml(filePath);
       fail("should fail");
     } catch (Throwable e) {
       assertTrue(e.getMessage().contains("file " + filePath + " is not found"));

+ 90 - 0
hadoop-hdds/common/src/test/java/org/apache/hadoop/hdds/scm/net/TestYamlSchemaLoader.java

@@ -0,0 +1,90 @@
+/**
+ * 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
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * 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.hdds.scm.net;
+
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.Timeout;
+import org.junit.runner.RunWith;
+import org.junit.runners.Parameterized;
+import org.junit.runners.Parameterized.Parameters;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.Arrays;
+import java.util.Collection;
+
+import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.fail;
+
+/** Test the node schema loader. */
+@RunWith(Parameterized.class)
+public class TestYamlSchemaLoader {
+  private static final Logger LOG =
+      LoggerFactory.getLogger(TestYamlSchemaLoader.class);
+  private ClassLoader classLoader =
+      Thread.currentThread().getContextClassLoader();
+
+  public TestYamlSchemaLoader(String schemaFile, String errMsg) {
+    try {
+      String filePath = classLoader.getResource(
+          "./networkTopologyTestFiles/" + schemaFile).getPath();
+      NodeSchemaLoader.getInstance().loadSchemaFromYaml(filePath);
+      fail("expect exceptions");
+    } catch (Throwable e) {
+      assertTrue(e.getMessage().contains(errMsg));
+    }
+  }
+
+  @Rule
+  public Timeout testTimeout = new Timeout(30000);
+
+  @Parameters
+  public static Collection<Object[]> getSchemaFiles() {
+    Object[][] schemaFiles = new Object[][]{
+        {"multiple-root.yaml", "Multiple root"},
+        {"middle-leaf.yaml", "Leaf node in the middle"},
+    };
+    return Arrays.asList(schemaFiles);
+  }
+
+
+  @Test
+  public void testGood() {
+    try {
+      String filePath = classLoader.getResource(
+              "./networkTopologyTestFiles/good.yaml").getPath();
+      NodeSchemaLoader.getInstance().loadSchemaFromYaml(filePath);
+    } catch (Throwable e) {
+      fail("should succeed");
+    }
+  }
+
+  @Test
+  public void testNotExist() {
+    String filePath = classLoader.getResource(
+        "./networkTopologyTestFiles/good.xml").getPath() + ".backup";
+    try {
+      NodeSchemaLoader.getInstance().loadSchemaFromXml(filePath);
+      fail("should fail");
+    } catch (Throwable e) {
+      assertTrue(e.getMessage().contains("file " + filePath + " is not found"));
+    }
+  }
+
+}

+ 59 - 0
hadoop-hdds/common/src/test/resources/networkTopologyTestFiles/good.yaml

@@ -0,0 +1,59 @@
+# 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
+#
+#     http://www.apache.org/licenses/LICENSE-2.0
+#
+# 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.
+
+---
+# Cost: The cost of crossing this layer.
+# The value should be positive integer or 0. This field is optional.
+# When it's not defined, it's value is default "1".
+cost: 1
+
+# The prefix of this layer.
+# If the prefix is "dc", then every name in this layer should start with "dc",
+# such as "dc1", "dc2".
+# Note that unlike XML schema, the prefix must be specified explicitly if the type is InnerNode.
+prefix: /
+
+# Layer type, optional field, default value InnerNode.
+# Current value range : {ROOT, INNER_NODE, LEAF_NODE}
+type: ROOT
+
+# Layer name
+defaultName: root
+
+# The sub layer of current layer. We use list
+sublayer:
+  -
+    cost: 1
+    prefix: dc
+    defaultName: datacenter
+    type: INNER_NODE
+    sublayer:
+      -
+        cost: 1
+        prefix: rack
+        defaultName: rack
+        type: INNER_NODE
+        sublayer:
+            -
+              cost: 1
+              prefix: ng
+              defaultName: nodegroup
+              type: INNER_NODE
+              sublayer:
+                -
+                  defaultName: node
+                  type: LEAF_NODE
+                  prefix: node
+...

+ 59 - 0
hadoop-hdds/common/src/test/resources/networkTopologyTestFiles/middle-leaf.yaml

@@ -0,0 +1,59 @@
+# 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
+#
+#     http://www.apache.org/licenses/LICENSE-2.0
+#
+# 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.
+
+---
+# Cost: The cost of crossing this layer.
+# The value should be positive integer or 0. This field is optional.
+# When it's not defined, it's value is default "1".
+cost: 1
+
+# The prefix of this layer.
+# If the prefix is "dc", then every name in this layer should start with "dc",
+# such as "dc1", "dc2".
+# Note that unlike XML schema, the prefix must be specified explicitly if the type is InnerNode.
+prefix: /
+
+# Layer type, optional field, default value InnerNode.
+# Current value range : {ROOT, INNER_NODE, LEAF_NODE}
+type: ROOT
+
+# Layer name
+defaultName: root
+
+# The sub layer of current layer. We use list
+sublayer:
+  -
+    cost: 1
+    prefix: dc
+    defaultName: datacenter
+    type: INNER_NODE
+    sublayer:
+      -
+        cost: 1
+        prefix: node
+        defaultName: rack
+        type: LEAF_NODE
+        sublayer:
+            -
+              cost: 1
+              prefix: ng
+              defaultName: nodegroup
+              type: INNER_NODE
+              sublayer:
+                -
+                  defaultName: node
+                  type: LEAF_NODE
+                  prefix: node
+...

+ 59 - 0
hadoop-hdds/common/src/test/resources/networkTopologyTestFiles/multiple-root.yaml

@@ -0,0 +1,59 @@
+# 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
+#
+#     http://www.apache.org/licenses/LICENSE-2.0
+#
+# 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.
+
+---
+# Cost: The cost of crossing this layer.
+# The value should be positive integer or 0. This field is optional.
+# When it's not defined, it's value is default "1".
+cost: 1
+
+# The prefix of this layer.
+# If the prefix is "dc", then every name in this layer should start with "dc",
+# such as "dc1", "dc2".
+# Note that unlike XML schema, the prefix must be specified explicitly if the type is InnerNode.
+prefix: /
+
+# Layer type, optional field, default value InnerNode.
+# Current value range : {ROOT, INNER_NODE, LEAF_NODE}
+type: ROOT
+
+# Layer name
+defaultName: root
+
+# The sub layer of current layer. We use list
+sublayer:
+  -
+    cost: 1
+    prefix: root
+    defaultName: root
+    type: ROOT
+    sublayer:
+      -
+        cost: 1
+        prefix: rack
+        defaultName: rack
+        type: INNER_NODE
+        sublayer:
+            -
+              cost: 1
+              prefix: ng
+              defaultName: nodegroup
+              type: INNER_NODE
+              sublayer:
+                -
+                  defaultName: node
+                  type: LEAF_NODE
+                  prefix: node
+...

+ 1 - 1
hadoop-hdds/container-service/pom.xml

@@ -48,7 +48,7 @@ http://maven.apache.org/xsd/maven-4.0.0.xsd">
     <dependency>
       <groupId>org.yaml</groupId>
       <artifactId>snakeyaml</artifactId>
-      <version>1.8</version>
+      <version>1.16</version>
     </dependency>
     <dependency>
       <groupId>com.google.code.findbugs</groupId>