Bläddra i källkod

HDDS-876. Add blockade tests for flaky network. Contributed by Nilotpal Nandi.

Mukul Kumar Singh 6 år sedan
förälder
incheckning
92d44b2ad0

+ 1 - 0
hadoop-ozone/dist/dev-support/bin/dist-layout-stitching

@@ -120,3 +120,4 @@ cp -r "${ROOT}/hadoop-hdds/docs/target/classes/docs" ./
 #Copy docker compose files
 run cp -p -r "${ROOT}/hadoop-ozone/dist/src/main/compose" .
 run cp -p -r "${ROOT}/hadoop-ozone/dist/src/main/smoketest" .
+run cp -p -r "${ROOT}/hadoop-ozone/dist/src/main/blockade" .

+ 26 - 0
hadoop-ozone/dist/src/main/blockade/README.md

@@ -0,0 +1,26 @@
+<!---
+  Licensed 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. See accompanying LICENSE file.
+-->
+
+## Blockade Tests
+Following python packages need to be installed before running the tests :
+
+1. blockade
+2. pytest==2.8.7
+
+You can execute the tests with following command-lines:
+
+```
+cd $DIRECTORY_OF_OZONE
+python -m pytest -s  blockade/
+```

+ 14 - 0
hadoop-ozone/dist/src/main/blockade/blockadeUtils/__init__.py

@@ -0,0 +1,14 @@
+# 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.

+ 59 - 0
hadoop-ozone/dist/src/main/blockade/blockadeUtils/blockade.py

@@ -0,0 +1,59 @@
+#!/usr/bin/python
+
+# 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.
+
+"""This module has apis to create and remove a blockade cluster"""
+
+from subprocess import call
+import subprocess
+import logging
+import random
+
+logger = logging.getLogger(__name__)
+
+
+class Blockade(object):
+
+    @classmethod
+    def blockade_destroy(cls):
+        call(["blockade", "destroy"])
+
+    @classmethod
+    def blockade_status(cls):
+        output = call(["blockade", "status"])
+        return output
+
+    @classmethod
+    def make_flaky(cls, flaky_node, container_list):
+        # make the network flaky
+        om = filter(lambda x: 'ozoneManager' in x, container_list)
+        scm = filter(lambda x: 'scm' in x, container_list)
+        datanodes = filter(lambda x: 'datanode' in x, container_list)
+        node_dict = {
+                "all": "--all",
+                "scm" : scm[0],
+                "om" : om[0],
+                "datanode": random.choice(datanodes)
+                }[flaky_node]
+        logger.info("flaky node: %s", node_dict)
+
+        output = call(["blockade", "flaky", node_dict])
+        assert output == 0, "flaky command failed with exit code=[%s]" % output
+
+    @classmethod
+    def blockade_fast_all(cls):
+        output = call(["blockade", "fast", "--all"])
+        assert output == 0, "fast command failed with exit code=[%s]" % output

+ 14 - 0
hadoop-ozone/dist/src/main/blockade/clusterUtils/__init__.py

@@ -0,0 +1,14 @@
+# 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.

+ 75 - 0
hadoop-ozone/dist/src/main/blockade/clusterUtils/cluster_utils.py

@@ -0,0 +1,75 @@
+#!/usr/bin/python
+
+# 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.
+
+"""This module has apis to create and remove a blockade cluster"""
+
+from subprocess import call
+import subprocess
+import logging
+import time
+
+
+logger = logging.getLogger(__name__)
+
+
+class ClusterUtils(object):
+
+    @classmethod
+    def cluster_setup(cls, docker_compose_file, datanode_count):
+        """start a blockade cluster"""
+        logger.info("compose file :%s", docker_compose_file)
+        logger.info("number of DNs :%d", datanode_count)
+        call(["docker-compose", "-f", docker_compose_file, "down"])
+        call(["docker-compose", "-f", docker_compose_file, "up", "-d", "--scale", "datanode=" + str(datanode_count)])
+
+        logger.info("Waiting 30s for cluster start up...")
+        time.sleep(30)
+        output = subprocess.check_output(["docker-compose", "-f", docker_compose_file, "ps"])
+        output_array = output.split("\n")[2:-1]
+
+        container_list = []
+        for out in output_array:
+            container = out.split(" ")[0]
+            container_list.append(container)
+            call(["blockade", "add", container])
+            time.sleep(2)
+
+        assert container_list, "no container found!"
+        logger.info("blockade created with containers %s", ' '.join(container_list))
+
+        return container_list
+
+    @classmethod
+    def cluster_destroy(cls, docker_compose_file):
+        call(["docker-compose", "-f", docker_compose_file, "down"])
+
+    @classmethod
+    def run_freon(cls, docker_compose_file, num_volumes, num_buckets, num_keys, key_size,
+                  replication_type, replication_factor):
+        # run freon
+        logger.info("Running freon ...")
+        output = call(["docker-compose", "-f", docker_compose_file,
+                                          "exec", "ozoneManager",
+                                          "/opt/hadoop/bin/ozone",
+                                          "freon", "rk",
+                                          "--numOfVolumes", str(num_volumes),
+                                          "--numOfBuckets", str(num_buckets),
+                                          "--numOfKeys", str(num_keys),
+                                          "--keySize", str(key_size),
+                                          "--replicationType", replication_type,
+                                          "--factor", replication_factor])
+        assert output == 0, "freon run failed with exit code=[%s]" % output

+ 65 - 0
hadoop-ozone/dist/src/main/blockade/conftest.py

@@ -0,0 +1,65 @@
+# 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.
+
+import logging
+import os
+
+
+def pytest_addoption(parser):
+    parser.addoption("--output-dir", action="store",
+                     default="/tmp/BlockadeTests",
+                     help="location of output directory where output log and plot files will be created")
+    parser.addoption("--log-format",
+                     action="store",
+                     default="%(asctime)s|%(levelname)s|%(threadName)s|%(filename)s:%(lineno)s -"
+                             " %(funcName)s()|%(message)s",
+                     help="specify log format")
+    parser.addoption("--log-level", action="store", default="info", help="specify log level")
+
+
+def pytest_configure(config):
+    outputdir = config.option.output_dir
+    try:
+        os.makedirs(outputdir)
+    except OSError, e:
+        raise Exception(e.strerror + ": " + e.filename)
+    log_file = os.path.join(outputdir, "output.log")
+
+    if config.option.log_level == "trace":
+        loglevel = eval("logging.DEBUG")
+    else:
+        loglevel = eval("logging." + config.option.log_level.upper())
+    logformatter = logging.Formatter(config.option.log_format)
+    logging.basicConfig(filename=log_file, filemode='w', level=loglevel, format=config.option.log_format)
+    console = logging.StreamHandler()
+    console.setLevel(loglevel)
+    console.setFormatter(logformatter)
+    logging.getLogger('').addHandler(console)
+
+
+def pytest_report_teststatus(report):
+    logger = logging.getLogger('main')
+    loc, line, name = report.location
+    if report.outcome == 'skipped':
+        pass
+    elif report.when == 'setup':
+        logger.info("RUNNING TEST \"%s\" at location \"%s\" at line number \"%s\"" % (name, loc, str(line)))
+    elif report.when == 'call':
+        logger.info("TEST \"%s\" %s in %3.2f seconds" % (name, report.outcome.upper(), report.duration))
+
+
+def pytest_sessionfinish(session):
+    logger = logging.getLogger('main')
+    logger.info("ALL TESTS FINISHED")

+ 54 - 0
hadoop-ozone/dist/src/main/blockade/test_blockade.py

@@ -0,0 +1,54 @@
+#!/usr/bin/python
+
+# 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.
+
+"""This module has apis to create and remove a blockade cluster"""
+import os
+import logging
+import pytest
+from blockadeUtils.blockade import Blockade
+from clusterUtils.cluster_utils import ClusterUtils
+
+
+logger = logging.getLogger(__name__)
+parent_dir = os.path.dirname(os.path.dirname(os.path.realpath(__file__)))
+FILE = os.path.join(parent_dir, "compose", "ozone", "docker-compose.yaml")
+SCALE = 6
+CONTAINER_LIST = []
+
+
+def setup_module():
+    global CONTAINER_LIST
+    CONTAINER_LIST = ClusterUtils.cluster_setup(FILE, SCALE)
+    output = Blockade.blockade_status()
+    assert output == 0, "blockade status command failed with exit code=[%s]" % output
+
+
+def teardown_module():
+    Blockade.blockade_destroy()
+    ClusterUtils.cluster_destroy(FILE)
+
+
+def teardown():
+    logger.info("Inside teardown")
+    Blockade.blockade_fast_all()
+
+
+@pytest.mark.parametrize("flaky_nodes", ["datanode", "scm", "om", "all"])
+def test_flaky(flaky_nodes):
+    Blockade.make_flaky(flaky_nodes, CONTAINER_LIST)
+    Blockade.blockade_status()
+    ClusterUtils.run_freon(FILE, 1, 1, 1, 10240, "RATIS", "THREE")