Skip to content

Commit

Permalink
Loosen environment validation checks for RCs (#28100)
Browse files Browse the repository at this point in the history
  • Loading branch information
tvalentyn authored Aug 22, 2023
1 parent 7a421c4 commit 6319d51
Show file tree
Hide file tree
Showing 2 changed files with 32 additions and 1 deletion.
13 changes: 12 additions & 1 deletion sdks/python/apache_beam/runners/worker/bundle_processor.py
Original file line number Diff line number Diff line change
Expand Up @@ -828,6 +828,17 @@ def only_element(iterable):
return element


def _environments_compatible(submission, runtime):
# type: (str, str) -> bool
if submission == runtime:
return True
if 'rc' in submission and runtime in submission:
# TODO(https://github.com/apache/beam/issues/28084): Loosen
# the check for RCs until RC containers install the matching version.
return True
return False


def _verify_descriptor_created_in_a_compatible_env(process_bundle_descriptor):
# type: (beam_fn_api_pb2.ProcessBundleDescriptor) -> None

Expand All @@ -836,7 +847,7 @@ def _verify_descriptor_created_in_a_compatible_env(process_bundle_descriptor):
env = process_bundle_descriptor.environments[t.environment_id]
for c in env.capabilities:
if (c.startswith(environments.SDK_VERSION_CAPABILITY_PREFIX) and
c != runtime_sdk):
not _environments_compatible(c, runtime_sdk)):
raise RuntimeError(
"Pipeline construction environment and pipeline runtime "
"environment are not compatible. If you use a custom "
Expand Down
20 changes: 20 additions & 0 deletions sdks/python/apache_beam/runners/worker/bundle_processor_test.py
Original file line number Diff line number Diff line change
Expand Up @@ -402,5 +402,25 @@ def test_can_sample_exceptions(self):
data_sampler.stop()


class EnvironmentCompatibilityTest(unittest.TestCase):
def test_rc_environments_are_compatible_with_released_images(self):
# TODO(https://github.com/apache/beam/issues/28084): remove when
# resolved.
self.assertTrue(
bundle_processor._environments_compatible(
"beam:version:sdk_base:apache/beam_python3.5_sdk:2.1.0rc1",
"beam:version:sdk_base:apache/beam_python3.5_sdk:2.1.0"))

def test_user_modified_sdks_need_to_be_installed_in_runtime_env(self):
self.assertFalse(
bundle_processor._environments_compatible(
"beam:version:sdk_base:apache/beam_python3.5_sdk:2.1.0-custom",
"beam:version:sdk_base:apache/beam_python3.5_sdk:2.1.0"))
self.assertTrue(
bundle_processor._environments_compatible(
"beam:version:sdk_base:apache/beam_python3.5_sdk:2.1.0-custom",
"beam:version:sdk_base:apache/beam_python3.5_sdk:2.1.0-custom"))


if __name__ == '__main__':
unittest.main()

0 comments on commit 6319d51

Please sign in to comment.