Skip to content

Commit f6b8d3e

Browse files
authored
Support Java 17+ compiled Beam components for build, release, and xlang tests (#35232)
* Install JDK 21 in release build Support Beam components requiring Java17+ for release workflows They will be compiled with JDK21 with byte code compatibility configured by applyJavaNature(requireJavaVersion) * Use Java 21 for Python PostCommit * Honor JAVA_HOME in JavaJarServer checks and tests * Disable Debezium test on Java17- * add example line
1 parent b4cae7f commit f6b8d3e

17 files changed

Lines changed: 97 additions & 20 deletions

File tree

.github/workflows/beam_PostCommit_Python.yml

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -78,6 +78,9 @@ jobs:
7878
- name: Setup environment
7979
uses: ./.github/actions/setup-environment-action
8080
with:
81+
java-version: |
82+
21
83+
11
8184
python-version: ${{ matrix.python_version }}
8285
- name: Install docker compose
8386
run: |
@@ -94,6 +97,7 @@ jobs:
9497
with:
9598
gradle-command: :python${{steps.set_py_ver_clean.outputs.py_ver_clean}}PostCommit
9699
arguments: |
100+
-Pjava21Home=$JAVA_HOME_21_X64 \
97101
-PuseWheelDistribution \
98102
-PpythonVersion=${{ matrix.python_version }} \
99103
env:

.github/workflows/beam_Release_NightlySnapshot.yml

Lines changed: 4 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -61,7 +61,9 @@ jobs:
6161
- name: Setup environment
6262
uses: ./.github/actions/setup-environment-action
6363
with:
64-
java-version: default
64+
java-version: |
65+
21
66+
11
6567
disable-cache: true
6668
- name: Auth on snapshot repository
6769
run: |
@@ -78,6 +80,7 @@ jobs:
7880
- name: run Publish script
7981
run: |
8082
./gradlew publish --max-workers=8 -Ppublishing -PskipCheckerFramework \
83+
-Pjava21Home=$JAVA_HOME_21_X64 \
8184
--continue -Dorg.gradle.jvmargs=-Xms2g -Dorg.gradle.jvmargs=-Xmx6g \
8285
-Dorg.gradle.vfs.watch=false -Pdocker-pull-licenses \
8386
-Dorg.gradle.internal.http.connectionTimeout=60000 \

.github/workflows/build_release_candidate.yml

Lines changed: 4 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -66,7 +66,9 @@ jobs:
6666
uses: actions/setup-java@v4
6767
with:
6868
distribution: 'temurin'
69-
java-version: '11'
69+
java-version: |
70+
21
71+
11
7072
- name: Import GPG key
7173
id: import_gpg
7274
uses: crazy-max/ghaction-import-gpg@111c56156bcc6918c056dbef52164cfa583dc549
@@ -92,7 +94,7 @@ jobs:
9294
- name: Configure git
9395
run: git config credential.helper store
9496
- name: Stage Java Artifacts into Maven
95-
run: ./gradlew publish -Psigning.gnupg.keyName=${{steps.import_gpg.outputs.fingerprint}} -PisRelease --no-daemon --no-parallel
97+
run: ./gradlew publish -Psigning.gnupg.keyName=${{steps.import_gpg.outputs.fingerprint}} -PisRelease -Pjava21Home=$JAVA_HOME_21_X64 --no-daemon --no-parallel
9698

9799

98100
stage_java_source:

runners/flink/job-server/flink_job_server.gradle

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -277,6 +277,12 @@ createCrossLanguageValidatesRunnerTask(
277277
],
278278
)
279279

280+
shadowJar {
281+
manifest {
282+
attributes(["Multi-Release": true])
283+
}
284+
}
285+
280286
// miniCluster jar starts an embedded Flink cluster intended for use in testing.
281287
tasks.register("miniCluster", Jar) {
282288
dependsOn shadowJar

sdks/python/apache_beam/io/external/xlang_debeziumio_it_test.py

Lines changed: 22 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -16,6 +16,7 @@
1616
#
1717

1818
import logging
19+
import subprocess
1920
import unittest
2021

2122
from apache_beam.io.debezium import DriverClassName
@@ -24,6 +25,7 @@
2425
from apache_beam.testing.test_pipeline import TestPipeline
2526
from apache_beam.testing.util import assert_that
2627
from apache_beam.testing.util import equal_to
28+
from apache_beam.utils import subprocess_server
2729

2830
# pylint: disable=wrong-import-order, wrong-import-position, ungrouped-imports
2931
try:
@@ -34,12 +36,32 @@
3436
NUM_RECORDS = 1
3537

3638

39+
def _disable_debezium_test():
40+
# disable if run on <Java17
41+
try:
42+
java = subprocess_server.JavaHelper.get_java()
43+
result = subprocess.run([java, '-version'],
44+
check=True,
45+
capture_output=True,
46+
text=True)
47+
version_line = result.stderr.splitlines()[0]
48+
# Example output: openjdk version "21.0.6" 2025-01-21
49+
version = version_line.split()[2].strip('\"')
50+
if int(version.split(".")[0]) < 17:
51+
return True
52+
except: # pylint: disable=bare-except
53+
return False
54+
55+
3756
@unittest.skipIf(
3857
PostgresContainer is None, 'testcontainers package is not installed')
3958
@unittest.skipIf(
4059
TestPipeline().get_pipeline_options().view_as(StandardOptions).runner
4160
is None,
4261
'Do not run this test on precommit suites.')
62+
@unittest.skipIf(
63+
_disable_debezium_test(),
64+
'Debezium test requires Java17+ in PATH or JAVA_HOME')
4365
class CrossLanguageDebeziumIOTest(unittest.TestCase):
4466
def setUp(self):
4567
self.username = 'debezium'

sdks/python/apache_beam/io/external/xlang_kafkaio_it_test.py

Lines changed: 8 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -40,6 +40,7 @@
4040
from apache_beam.testing.util import equal_to
4141
from apache_beam.transforms.userstate import BagStateSpec
4242
from apache_beam.transforms.userstate import CombiningValueStateSpec
43+
from apache_beam.utils import subprocess_server
4344

4445
NUM_RECORDS = 1000
4546

@@ -220,8 +221,13 @@ def local_kafka_service(self, local_kafka_jar_file):
220221
zookeeper_port = str(self.get_open_port())
221222
kafka_server = None
222223
try:
223-
kafka_server = subprocess.Popen(
224-
['java', '-jar', local_kafka_jar_file, kafka_port, zookeeper_port])
224+
kafka_server = subprocess.Popen([
225+
subprocess_server.JavaHelper.get_java(),
226+
'-jar',
227+
local_kafka_jar_file,
228+
kafka_port,
229+
zookeeper_port
230+
])
225231
time.sleep(3)
226232
yield kafka_port
227233
finally:

sdks/python/apache_beam/io/kafka.py

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -45,7 +45,8 @@
4545
pipeline.
4646
4747
* Install Java runtime in the computer from where the pipeline is constructed
48-
and make sure that 'java' command is available.
48+
and make sure that 'java' command is available or set JAVA_HOME environment
49+
variable.
4950
5051
In this option, Python SDK will either download (for released Beam version) or
5152
build (when running from a Beam Git clone) an expansion service jar and use

sdks/python/apache_beam/io/kinesis.py

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -44,7 +44,8 @@
4444
pipeline.
4545
4646
* Install Java runtime in the computer from where the pipeline is constructed
47-
and make sure that 'java' command is available.
47+
and make sure that 'java' command is available or set JAVA_HOME environment
48+
variable.
4849
4950
In this option, Python SDK will either download (for released Beam version) or
5051
build (when running from a Beam Git clone) a expansion service jar and use

sdks/python/apache_beam/io/snowflake.py

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -39,7 +39,8 @@
3939
before running the Beam pipeline.
4040
4141
* Install Java runtime in the computer from where the pipeline is constructed
42-
and make sure that 'java' command is available.
42+
and make sure that 'java' command is available or set JAVA_HOME environment
43+
variable.
4344
4445
In this option, Python SDK will either download (for released Beam version) or
4546
build (when running from a Beam Git clone) a expansion service jar and use

sdks/python/apache_beam/runners/portability/samza_runner_test.py

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -33,6 +33,7 @@
3333
from apache_beam.runners.portability import job_server
3434
from apache_beam.runners.portability import portable_runner
3535
from apache_beam.runners.portability import portable_runner_test
36+
from apache_beam.utils import subprocess_server
3637

3738
_LOGGER = logging.getLogger(__name__)
3839

@@ -97,7 +98,7 @@ def _subprocess_command(cls, job_port, expansion_port):
9798

9899
try:
99100
return [
100-
'java',
101+
subprocess_server.JavaHelper.get_java(),
101102
'-jar',
102103
cls.samza_job_server_jar,
103104
'--artifacts-dir',

0 commit comments

Comments
 (0)