Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Fix stateful processing using direct runner with type checks enabled #27646

Merged
merged 8 commits into from
Aug 10, 2023

Conversation

sadovnychyi
Copy link
Contributor

@sadovnychyi sadovnychyi commented Jul 24, 2023

With this change all of the public methods of DoFns are now properly proxied from AbstractDoFnWrapper, which fixes an issue with using stateful DoFns on direct runner with runtime type checks enabled.

Closes #27167


Thank you for your contribution! Follow this checklist to help us incorporate your contribution quickly and easily:

  • Mention the appropriate issue in your description (for example: addresses #123), if applicable. This will automatically add a link to the pull request in the issue. If you would like the issue to automatically close on merging the pull request, comment fixes #<ISSUE NUMBER> instead.
  • Update CHANGES.md with noteworthy changes.
  • If this contribution is large, please file an Apache Individual Contributor License Agreement.

See the Contributor Guide for more tips on how to make review process smoother.

To check the build health, please visit https://github.com/apache/beam/blob/master/.test-infra/BUILD_STATUS.md

GitHub Actions Tests Status (on master branch)

Build python source distribution and wheels
Python tests
Java tests
Go tests

See CI.md for more information about GitHub Actions CI.

@sadovnychyi
Copy link
Contributor Author

R: @tvalentyn

@github-actions
Copy link
Contributor

Stopping reviewer notifications for this pull request: review requested by someone other than the bot, ceding control

@codecov
Copy link

codecov bot commented Jul 24, 2023

Codecov Report

Merging #27646 (81cd93a) into master (c654dc0) will decrease coverage by 0.01%.
Report is 61 commits behind head on master.
The diff coverage is 100.00%.

@@            Coverage Diff             @@
##           master   #27646      +/-   ##
==========================================
- Coverage   70.87%   70.87%   -0.01%     
==========================================
  Files         861      861              
  Lines      105001   105005       +4     
==========================================
+ Hits        74421    74422       +1     
- Misses      29022    29025       +3     
  Partials     1558     1558              
Flag Coverage Δ
python 79.83% <100.00%> (-0.01%) ⬇️

Flags with carried forward coverage won't be shown. Click here to find out more.

Files Changed Coverage Δ
sdks/python/apache_beam/typehints/typecheck.py 97.66% <100.00%> (+0.04%) ⬆️

... and 5 files with indirect coverage changes

📣 We’re building smart automated test selection to slash your CI/CD build times. Learn more

options = PipelineOptions()
options.view_as(TypeOptions).runtime_type_check = True
with TestPipeline(options=options) as pipeline:
collection = pipeline \
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

nit / personal opinion: you can use brackets instead of line continuation tokens

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Love it, done.

YAPF seems to be way behind black in this regard, I think I remember seeing somewhere you folks talking about migrating to black, was it a dead end?

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

We're still on yapf as far as our style infra, idk if anyone is actively investigating alternatives right now

sdks/python/apache_beam/transforms/util_test.py Outdated Show resolved Hide resolved
@tvalentyn
Copy link
Contributor

Run Portable_Python PreCommit

@tvalentyn
Copy link
Contributor

Upon closer look, it seems that current design would still be prone to similar gaps.

Also would appreciate if anybody can suggest something better than hasattr check.

I wonder if overriding AbstractDoFnWrapper__getattr__ would help:

def __getattr__(self, attr):
  return getattr(self.dofn, attr)

@sadovnychyi
Copy link
Contributor Author

I wonder if overriding AbstractDoFnWrapper__getattr__ would help:

It falls into a recursion like so:

  File "/Users/sadovnychyi/github.com/beam/sdks/python/apache_beam/runners/direct/transform_evaluator.py", line 834, in start_bundle
    pickler.loads(pickler.dumps(transform.dofn))
  File "/Users/sadovnychyi/github.com/beam/sdks/python/apache_beam/internal/pickler.py", line 51, in loads
    return desired_pickle_lib.loads(
           ^^^^^^^^^^^^^^^^^^^^^^^^^
  File "/Users/sadovnychyi/github.com/beam/sdks/python/apache_beam/internal/dill_pickler.py", line 422, in loads
    return dill.loads(s)
           ^^^^^^^^^^^^^
  File "/Users/sadovnychyi/github.com/beam/.venv/lib/python3.11/site-packages/dill/_dill.py", line 275, in loads
    return load(file, ignore, **kwds)
           ^^^^^^^^^^^^^^^^^^^^^^^^^^
  File "/Users/sadovnychyi/github.com/beam/.venv/lib/python3.11/site-packages/dill/_dill.py", line 270, in load
    return Unpickler(file, ignore=ignore, **kwds).load()
           ^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^^
  File "/Users/sadovnychyi/github.com/beam/.venv/lib/python3.11/site-packages/dill/_dill.py", line 472, in load
    obj = StockUnpickler.load(self)
          ^^^^^^^^^^^^^^^^^^^^^^^^^
  File "/Users/sadovnychyi/github.com/beam/sdks/python/apache_beam/typehints/typecheck.py", line 57, in __getattr__
    return getattr(self.dofn, attr)
                   ^^^^^^^^^
  File "/Users/sadovnychyi/github.com/beam/sdks/python/apache_beam/typehints/typecheck.py", line 57, in __getattr__
    return getattr(self.dofn, attr)
                   ^^^^^^^^^
  File "/Users/sadovnychyi/github.com/beam/sdks/python/apache_beam/typehints/typecheck.py", line 57, in __getattr__
    return getattr(self.dofn, attr)
                   ^^^^^^^^^
  [Previous line repeated 2985 more times]
RecursionError: maximum recursion depth exceeded while calling a Python object

----------------------------------------------------------------------
Ran 8 tests in 2.809s

FAILED (errors=1)

@tvalentyn
Copy link
Contributor

there is also a better example than what I suggested:

def __getattribute__(self, name):
, you could try that.

@sadovnychyi
Copy link
Contributor Author

there is also a better example than what I suggested:

def __getattribute__(self, name):

, you could try that.

Thank you! It worked with a tiny change because of inheritance. Will see whether this passes the whole test suite.

@jrmccluskey
Copy link
Contributor

The unit test that is failing consistently - https://github.com/apache/beam/blob/master/sdks/python/apache_beam/transforms/ptransform_test.py#L2043 - getting TypeError: sequence item 0: expected str instance, int found instead of a typehints.TypeCheckError with the content Runtime type violation detected within ParDo(SortJoin/KeyWithVoid): Type-hint for argument: 'v' violated. Expected an instance of str, instead found 0, an instance of int.

So it looks like we're hitting some issues with this type mismatch getting surfaced incorrectly with the change here.

Copy link
Contributor

@jrmccluskey jrmccluskey left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Unit test must be passing before merge

@tvalentyn
Copy link
Contributor

Looks like tests are passing. for my education, why was the Don't proxy internal methods into dofn commit necessary?

@tvalentyn
Copy link
Contributor

also please update the PR description.

@@ -49,6 +49,13 @@ def __init__(self, dofn):
super().__init__()
self.dofn = dofn

def __getattribute__(self, name):
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

also we should now be able to drop setup and teardown overrides.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

With this change the setup and teardown are no longer called in here:

def test_wrapper_pass_through(self):
# We use a file to check the result because the MyDoFn instance passed is
# not the same one that actually runs in the pipeline (it is serialized
# here and deserialized in the worker).
with tempfile.TemporaryDirectory() as tmp_dirname:
path = os.path.join(tmp_dirname + "tmp_filename")
dofn = MyDoFn(path)
result = self.p | beam.Create([1, 2, 3]) | beam.ParDo(dofn)
assert_that(result, equal_to([1, 2, 3]))
self.p.run()
with open(path, mode="r") as ft:
lines = [line.strip() for line in ft]
self.assertListEqual([
'setup',
'start_bundle',
'process',
'process',
'process',
'finish_bundle',
'teardown',
],
lines)

Reverted for now: 81cd93a

Any way we can proceed as is? Otherwise please suggest where to start digging, thanks!

@sadovnychyi
Copy link
Contributor Author

Looks like tests are passing. for my education, why was the Don't proxy internal methods into dofn commit necessary?

This was the easiest fix for test_combine_runtime_type_check_violation_using_methods.

also please update the PR description.

Done!

Not sure if it makes sense to repeat this check.
Black-style, YAPF doesn't mind.
This makes it more future proof.
Inspired by `_ExceptionHandlingWrapperDoFn` but with `type(self)` so that
it works with inheritance.
This fixes the test_combine_runtime_type_check_violation_using_methods.
Each is now covered by `__getattribute__`.
@sadovnychyi
Copy link
Contributor Author

I see that some tests are failing now -- I'll take a look next week.

This reverts commit 27f67df.

Coudln't find a way to keep the `RuntimeTypeCheckTest.test_wrapper_pass_through`
working except for the following patch:

```diff
diff --git a/sdks/python/apache_beam/typehints/typecheck.py b/sdks/python/apache_beam/typehints/typecheck.py
index 6d4b4d4962..31d8828fb6 100644
--- a/sdks/python/apache_beam/typehints/typecheck.py
+++ b/sdks/python/apache_beam/typehints/typecheck.py
@@ -51,7 +51,7 @@ class AbstractDoFnWrapper(DoFn):

   def __getattribute__(self, name):
     if (name.startswith('_') or name in self.__dict__ or
-        hasattr(type(self), name)):
+        hasattr(type(self), name)) and name not in ('setup', 'teardown'):
       return object.__getattribute__(self, name)
     else:
       return getattr(self.dofn, name)
```

In which case we might as well keep the methods.
@tvalentyn
Copy link
Contributor

Run Portable_Python PreCommit

@tvalentyn
Copy link
Contributor

Run Python_PVR_Flink PreCommit

@tvalentyn
Copy link
Contributor

thanks! rerunning remaining suites, feel free to ping this PR if not merged after tests pass.

@tvalentyn tvalentyn merged commit 6f60a6c into apache:master Aug 10, 2023
76 checks passed
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment
Labels
Projects
None yet
Development

Successfully merging this pull request may close these issues.

[Bug]: runtime_type_check breaks stateful processing with timers
3 participants