This is an automated email from the ASF dual-hosted git repository. damccorm pushed a commit to branch users/damccorm/revertTFService in repository https://gitbox.apache.org/repos/asf/beam.git
commit db98a025ad9cf119ccf8f64119bc44f6aed6c2a4 Author: Danny McCormick <dannymccorm...@google.com> AuthorDate: Thu Aug 17 12:09:07 2023 -0400 Revert transform service changes --- .../job_PostCommit_TransformService_Direct.groovy | 53 --------- .../org/apache/beam/gradle/BeamModulePlugin.groovy | 126 --------------------- .../launcher/TransformServiceLauncher.java | 15 ++- .../apache_beam/io/gcp/bigtableio_it_test.py | 2 - sdks/python/pytest.ini | 1 - sdks/python/scripts/run_transform_service.sh | 85 -------------- sdks/python/test-suites/direct/xlang/build.gradle | 19 ---- 7 files changed, 7 insertions(+), 294 deletions(-) diff --git a/.test-infra/jenkins/job_PostCommit_TransformService_Direct.groovy b/.test-infra/jenkins/job_PostCommit_TransformService_Direct.groovy deleted file mode 100644 index 0d7f58e7170..00000000000 --- a/.test-infra/jenkins/job_PostCommit_TransformService_Direct.groovy +++ /dev/null @@ -1,53 +0,0 @@ -/* - * 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 CommonJobProperties as commonJobProperties -import PostcommitJobBuilder - -import static PythonTestProperties.CROSS_LANGUAGE_VALIDATES_RUNNER_PYTHON_VERSIONS - -// This job runs multi-language pipelines using the Docker Compose based TransformService against the Direct runner. -// Collects tests with the @pytest.mark.uses_transform_service decorator -PostcommitJobBuilder.postCommitJob('beam_PostCommit_TransformService_Direct', - 'Run TransformService_Direct PostCommit', 'Direct TransformService Tests', this) { - description('Runs the TransformService suite on the Direct runner.') - - // Set common parameters. - commonJobProperties.setTopLevelMainJobProperties(delegate, 'master', 120) - - // Publish all test results to Jenkins - publishers { - archiveJunit('**/pytest*.xml') - } - - // Gradle goals for this job. - steps { - CROSS_LANGUAGE_VALIDATES_RUNNER_PYTHON_VERSIONS.each { pythonVersion -> - shell("echo \"*** RUN TRANSFORM SERVICE Python SDK TESTS USING THE DIRECT RUNNER AND THE PYTHON VERSION ${pythonVersion} ***\"") - gradle { - rootBuildScriptDir(commonJobProperties.checkoutDir) - tasks(':sdks:python:test-suites:direct:xlang:transformServicePythonUsingJava') - commonJobProperties.setGradleSwitches(delegate) - switches '-PcompileAndRunTestsWithJava11' - switches "-Pjava11Home=${commonJobProperties.JAVA_11_HOME}" - switches("-PuseWheelDistribution") - switches("-PpythonVersion=${pythonVersion}") - } - } - } - } diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index b3c4b4949ab..81b7a4f965a 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -402,30 +402,6 @@ class BeamModulePlugin implements Plugin<Project> { FileCollection classpath } - // A class defining the configuration for createTransformServiceTask. - static class TransformServiceConfiguration { - // Task name TransformService case. - String name = 'transformService' - - List<String> pythonPipelineOptions = [] - - List<String> javaPipelineOptions = [] - - // Additional pytest options - List<String> pytestOptions = [] - // Job server startup task. - TaskProvider startJobServer - // Job server cleanup task. - TaskProvider cleanupJobServer - // Number of parallel test runs. - Integer numParallelTests = 1 - // Whether the pipeline needs --sdk_location option - boolean needsSdkLocation = false - - // Collect Python pipeline tests with this marker - String collectMarker - } - def isRelease(Project project) { return parseBooleanProperty(project, 'isRelease'); } @@ -2765,108 +2741,6 @@ class BeamModulePlugin implements Plugin<Project> { /** ***********************************************************************************************/ - // Method to create the createTransformServiceTask. - // The method takes TransformServiceConfiguration as parameter. - project.ext.createTransformServiceTask = { - // This task won't work if the python build file doesn't exist. - if (!project.project(":sdks:python").buildFile.exists()) { - System.err.println 'Python build file not found. Skipping createTransformServiceTask.' - return - } - def config = it ? it as TransformServiceConfiguration : new TransformServiceConfiguration() - - project.evaluationDependsOn(":sdks:python") - project.evaluationDependsOn(":runners:core-construction-java") - project.evaluationDependsOn(":sdks:java:extensions:python") - project.evaluationDependsOn(":sdks:java:transform-service:launcher") - - def usesDataflowRunner = config.pythonPipelineOptions.contains("--runner=TestDataflowRunner") || config.pythonPipelineOptions.contains("--runner=DataflowRunner") - - // Task for launching transform services - def envDir = project.project(":sdks:python").envdir - def pythonDir = project.project(":sdks:python").projectDir - def externalPort = getRandomPort() - def launcherJar = project.project(':sdks:java:transform-service:launcher').shadowJar.archivePath - def transformServiceOpts = [ - "transform_service_launcher_jar": launcherJar, - "group_id": project.name, - "external_port": externalPort, - "beam_version": project.version - ] - def serviceArgs = project.project(':sdks:python').mapToArgString(transformServiceOpts) - def pythonContainerSuffix = project.project(':sdks:python').pythonVersion.replace('.', '') - def javaContainerSuffix - if (JavaVersion.current() == JavaVersion.VERSION_1_8) { - javaContainerSuffix = 'java8' - } else if (JavaVersion.current() == JavaVersion.VERSION_11) { - javaContainerSuffix = 'java11' - } else if (JavaVersion.current() == JavaVersion.VERSION_17) { - javaContainerSuffix = 'java17' - } else { - String exceptionMessage = "Your Java version is unsupported. You need Java version of 8 or 11 or 17 to get started, but your Java version is: " + JavaVersion.current(); - throw new GradleException(exceptionMessage) - } - - def setupTask = project.tasks.register(config.name+"Setup", Exec) { - dependsOn ':sdks:java:container:'+javaContainerSuffix+':docker' - dependsOn ':sdks:python:container:py'+pythonContainerSuffix+':docker' - dependsOn ':sdks:java:transform-service:controller-container:docker' - dependsOn ':sdks:python:expansion-service-container:docker' - dependsOn ':sdks:java:expansion-service:container:docker' - dependsOn ":sdks:python:installGcpTest" - dependsOn project.project(':sdks:java:transform-service:launcher').shadowJar.getPath() - - if (usesDataflowRunner) { - dependsOn ":sdks:python:test-suites:dataflow:py${project.ext.pythonVersion.replace('.', '')}:initializeForDataflowJob" - } - - // setup test env - executable 'sh' - args '-c', "$pythonDir/scripts/run_transform_service.sh stop $serviceArgs && $pythonDir/scripts/run_transform_service.sh start $serviceArgs" - } - - if (config.needsSdkLocation) { - setupTask.configure {dependsOn ':sdks:python:sdist'} - } - - def pythonTask = project.tasks.register(config.name+"PythonUsingJava") { - group = "Verification" - description = "Runs Python SDK pipeline tests that use transform service" - dependsOn setupTask - dependsOn config.startJobServer - doLast { - def beamPythonTestPipelineOptions = [ - "pipeline_opts": config.pythonPipelineOptions + (usesDataflowRunner ? [ - "--sdk_location=${project.ext.sdkLocation}"] - : []), - "test_opts": config.pytestOptions, - "suite": config.name, - "collect": config.collectMarker, - ] - def cmdArgs = project.project(':sdks:python').mapToArgString(beamPythonTestPipelineOptions) - - project.exec { - environment "EXPANSION_PORT", externalPort - executable 'sh' - args '-c', ". $envDir/bin/activate && cd $pythonDir && ./scripts/run_integration_test.sh $cmdArgs" - } - } - } - - def cleanupTask = project.tasks.register(config.name+'Cleanup', Exec) { - // teardown test env - executable 'sh' - args '-c', "$pythonDir/scripts/run_transform_service.sh stop $serviceArgs" - } - setupTask.configure {finalizedBy cleanupTask} - config.startJobServer.configure {finalizedBy config.cleanupJobServer} - - cleanupTask.configure{mustRunAfter pythonTask} - config.cleanupJobServer.configure{mustRunAfter pythonTask} - } - - /** ***********************************************************************************************/ - project.ext.applyPythonNature = { // Define common lifecycle tasks and artifact types diff --git a/sdks/java/transform-service/launcher/src/main/java/org/apache/beam/sdk/transformservice/launcher/TransformServiceLauncher.java b/sdks/java/transform-service/launcher/src/main/java/org/apache/beam/sdk/transformservice/launcher/TransformServiceLauncher.java index 3878c42affe..dbe9b7857fa 100644 --- a/sdks/java/transform-service/launcher/src/main/java/org/apache/beam/sdk/transformservice/launcher/TransformServiceLauncher.java +++ b/sdks/java/transform-service/launcher/src/main/java/org/apache/beam/sdk/transformservice/launcher/TransformServiceLauncher.java @@ -28,7 +28,6 @@ import java.util.List; import java.util.Locale; import java.util.Map; import java.util.concurrent.TimeoutException; -import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.collect.ImmutableList; import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.io.ByteStreams; import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.io.Files; import org.checkerframework.checker.nullness.qual.Nullable; @@ -155,15 +154,15 @@ public class TransformServiceLauncher { return launchers.get(projectName); } - private void runDockerComposeCommand(List<String> command) throws IOException { + private void runDockerComposeCommand(String command) throws IOException { this.runDockerComposeCommand(command, null); } - private void runDockerComposeCommand(List<String> command, @Nullable File outputOverride) + private void runDockerComposeCommand(String command, @Nullable File outputOverride) throws IOException { List<String> shellCommand = new ArrayList<>(); shellCommand.addAll(dockerComposeStartCommandPrefix); - shellCommand.addAll(command); + shellCommand.add(command); System.out.println("Executing command: " + String.join(" ", command)); ProcessBuilder processBuilder = new ProcessBuilder(shellCommand).redirectError(ProcessBuilder.Redirect.INHERIT); @@ -187,15 +186,15 @@ public class TransformServiceLauncher { } public synchronized void start() throws IOException, TimeoutException { - runDockerComposeCommand(ImmutableList.of("up", "-d")); + runDockerComposeCommand("up"); } public synchronized void shutdown() throws IOException { - runDockerComposeCommand(ImmutableList.of("down")); + runDockerComposeCommand("down"); } public synchronized void status() throws IOException { - runDockerComposeCommand(ImmutableList.of("ps")); + runDockerComposeCommand("ps"); } public synchronized void waitTillUp(int timeout) throws IOException, TimeoutException { @@ -226,7 +225,7 @@ public class TransformServiceLauncher { private synchronized String getStatus() throws IOException { File outputOverride = File.createTempFile("output_override", null); - runDockerComposeCommand(ImmutableList.of("ps"), outputOverride); + runDockerComposeCommand("ps", outputOverride); return outputOverride.getAbsolutePath(); } diff --git a/sdks/python/apache_beam/io/gcp/bigtableio_it_test.py b/sdks/python/apache_beam/io/gcp/bigtableio_it_test.py index 341f2983c8b..4c26da7012d 100644 --- a/sdks/python/apache_beam/io/gcp/bigtableio_it_test.py +++ b/sdks/python/apache_beam/io/gcp/bigtableio_it_test.py @@ -50,7 +50,6 @@ except ImportError as e: @pytest.mark.uses_gcp_java_expansion_service -@pytest.mark.uses_transform_service @unittest.skipUnless( os.environ.get('EXPANSION_PORT'), "EXPANSION_PORT environment var is not provided.") @@ -143,7 +142,6 @@ class TestReadFromBigTableIT(unittest.TestCase): @pytest.mark.uses_gcp_java_expansion_service -@pytest.mark.uses_transform_service @unittest.skipUnless( os.environ.get('EXPANSION_PORT'), "EXPANSION_PORT environment var is not provided.") diff --git a/sdks/python/pytest.ini b/sdks/python/pytest.ini index ec8d9fb3900..43db269a80a 100644 --- a/sdks/python/pytest.ini +++ b/sdks/python/pytest.ini @@ -31,7 +31,6 @@ markers = uses_java_expansion_service: collect Cross Language Java transforms test runs uses_python_expansion_service: collect Cross Language Python transforms test runs uses_io_expansion_service: collect Cross Language transform test runs (with Kafka bootstrap server) - uses_transform_service: collect Cross Language test runs that uses the Transform Service xlang_sql_expansion_service: collect for Cross Language with SQL expansion service test runs it_postcommit: collect for post-commit integration test runs it_postcommit_sickbay: collect for post-commit sickbay integration test run diff --git a/sdks/python/scripts/run_transform_service.sh b/sdks/python/scripts/run_transform_service.sh deleted file mode 100755 index d3bf12158d7..00000000000 --- a/sdks/python/scripts/run_transform_service.sh +++ /dev/null @@ -1,85 +0,0 @@ -#!/bin/bash -# -# 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. -# - -read -r -d '' USAGE <<END -Usage: run_expansion_services.sh (start|stop) [options] -Options: - --group_id [unique id for stop services later] - --transform_service_launcher_jar [path to the transform service launcher jar] - --external_port [external port exposed by the transform service] - --start [command to start the transform service for the given group_id] - --stop [command to stop the transform service for the given group_id] -END - -while [[ $# -gt 0 ]]; do - key="$1" - case $key in - --group_id) - GROUP_ID="$2" - shift - shift - ;; - --transform_service_launcher_jar) - TRANSFORM_SERVICE_LAUNCHER_JAR="$2" - shift - shift - ;; - --external_port) - EXTERNAL_PORT="$2" - shift - shift - ;; - --beam_version) - BEAM_VERSION_JAR="$2" - BEAM_VERSION_DOCKER=${BEAM_VERSION_JAR/-SNAPSHOT/.dev} - shift - shift - ;; - start) - STARTSTOP="$1" - shift - ;; - stop) - STARTSTOP="$1" - shift - ;; - *) - echo "Unknown option: $1" - echo "$USAGE" - exit 1 - ;; - esac -done - -FILE_BASE="beam-transform-service" -if [ -n "$GROUP_ID" ]; then - FILE_BASE="$FILE_BASE-$GROUP_ID" -fi - -TEMP_DIR=/tmp - -case $STARTSTOP in - start) - echo "Starting the transform service for project $GROUP_ID at port $EXTERNAL_PORT for Beam version $BEAM_VERSION_DOCKER transform service startup jar is $TRANSFORM_SERVICE_LAUNCHER_JAR" - java -jar $TRANSFORM_SERVICE_LAUNCHER_JAR --project_name $GROUP_ID --port $EXTERNAL_PORT --beam_version $BEAM_VERSION_DOCKER --command up >$TEMP_DIR/$FILE_BASE-java1.log 2>&1 </dev/null & - ;; - stop) - echo "Stopping the transform service for project $GROUP_ID at port $EXTERNAL_PORT for Beam version $BEAM_VERSION_DOCKER transform service startup jar is $TRANSFORM_SERVICE_LAUNCHER_JAR" - java -jar $TRANSFORM_SERVICE_LAUNCHER_JAR --project_name $GROUP_ID --port $EXTERNAL_PORT --beam_version $BEAM_VERSION_DOCKER --command down >$TEMP_DIR/$FILE_BASE-java2.log 2>&1 </dev/null & - ;; -esac diff --git a/sdks/python/test-suites/direct/xlang/build.gradle b/sdks/python/test-suites/direct/xlang/build.gradle index 289f5c8a0e0..74cda691f14 100644 --- a/sdks/python/test-suites/direct/xlang/build.gradle +++ b/sdks/python/test-suites/direct/xlang/build.gradle @@ -44,8 +44,6 @@ def cleanupTask = project.tasks.register("fnApiJobServerCleanup", Exec) { args '-c', ". ${envDir}/bin/activate && python -m apache_beam.runners.portability.local_job_service_main --pid_file ${pidFile} --stop" } -def gcpProject = project.findProperty('dataflowProject') ?: 'apache-beam-testing' - createCrossLanguageValidatesRunnerTask( startJobServer: setupTask, cleanupJobServer: cleanupTask, @@ -69,20 +67,3 @@ createCrossLanguageValidatesRunnerTask( "--endpoint localhost:${jobPort}", ], ) - -createTransformServiceTask( - startJobServer: setupTask, - cleanupJobServer: cleanupTask, - numParallelTests: 1, - collectMarker: 'uses_transform_service', - pythonPipelineOptions: [ - "--runner=TestDirectRunner", - "--project=${gcpProject}", - ], - pytestOptions: [ - "--capture=no", // print stdout instantly - "--timeout=4500", // timeout of whole command execution - "--color=yes", // console color - "--log-cli-level=INFO" //log level info - ] -)