Skip to content

Commit

Permalink
Merge pull request #32839 Use WindowedValueParam for interactive cache.
Browse files Browse the repository at this point in the history
  • Loading branch information
robertwb authored Oct 18, 2024
2 parents ef0fc8b + 418a1d4 commit 5d0dcfc
Showing 1 changed file with 2 additions and 8 deletions.
10 changes: 2 additions & 8 deletions sdks/python/apache_beam/runners/interactive/caching/reify.py
Original file line number Diff line number Diff line change
Expand Up @@ -28,7 +28,6 @@
import apache_beam as beam
from apache_beam.runners.interactive import cache_manager as cache
from apache_beam.testing import test_stream
from apache_beam.transforms.window import WindowedValue

READ_CACHE = 'ReadCache_'
WRITE_CACHE = 'WriteCache_'
Expand All @@ -40,13 +39,8 @@ class Reify(beam.DoFn):
Internally used to capture window info with each element into cache for
replayability.
"""
def process(
self,
e,
w=beam.DoFn.WindowParam,
p=beam.DoFn.PaneInfoParam,
t=beam.DoFn.TimestampParam):
yield test_stream.WindowedValueHolder(WindowedValue(e, t, [w], p))
def process(self, e, wv=beam.DoFn.WindowedValueParam):
yield test_stream.WindowedValueHolder(wv)


class Unreify(beam.DoFn):
Expand Down

0 comments on commit 5d0dcfc

Please sign in to comment.