[ https://issues.apache.org/jira/browse/BEAM-1440?focusedWorklogId=352622&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-352622 ]
ASF GitHub Bot logged work on BEAM-1440: ---------------------------------------- Author: ASF GitHub Bot Created on: 03/Dec/19 13:49 Start Date: 03/Dec/19 13:49 Worklog Time Spent: 10m Work Description: kamilwu commented on pull request #9772: [BEAM-1440] Create a BigQuery source that implements iobase.BoundedSource for Python URL: https://github.com/apache/beam/pull/9772#discussion_r353187727 ########## File path: sdks/python/apache_beam/io/gcp/bigquery.py ########## @@ -1268,3 +1461,140 @@ def display_data(self): tableSpec) res['table'] = DisplayDataItem(tableSpec, label='Table') return res + + +@experimental() +class PassThroughThenCleanup(PTransform): + """A PTransform that invokes a DoFn after the input PCollection has been + processed. + """ + def __init__(self, cleanup_dofn): + self.cleanup_dofn = cleanup_dofn + + def expand(self, input): + class PassThrough(beam.DoFn): + def process(self, element): + yield element + + output = input | beam.ParDo(PassThrough()).with_outputs('cleanup_signal', + main='main') + main_output = output['main'] + cleanup_signal = output['cleanup_signal'] + + _ = (input.pipeline + | beam.Create([None]) + | beam.ParDo(self.cleanup_dofn, beam.pvalue.AsSingleton( + cleanup_signal))) + + return main_output + + +@experimental() +class ReadFromBigQuery(PTransform): Review comment: I'm totally fine with it. Once this PR is merged, I'm going to make changes to the Chicago Taxi Example, so that the example would use this transform. That would be a great opportunity to check stability and measure performance of the transform ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org Issue Time Tracking ------------------- Worklog Id: (was: 352622) Time Spent: 11h 40m (was: 11.5h) > Create a BigQuery source (that implements iobase.BoundedSource) for Python SDK > ------------------------------------------------------------------------------ > > Key: BEAM-1440 > URL: https://issues.apache.org/jira/browse/BEAM-1440 > Project: Beam > Issue Type: New Feature > Components: sdk-py-core > Reporter: Chamikara Madhusanka Jayalath > Assignee: Kamil Wasilewski > Priority: Major > Time Spent: 11h 40m > Remaining Estimate: 0h > > Currently we have a BigQuery native source for Python SDK [1]. > This can only be used by Dataflow runner. > We should implement a Beam BigQuery source that implements > iobase.BoundedSource [2] interface so that other runners that try to use > Python SDK can read from BigQuery as well. Java SDK already has a Beam > BigQuery source [3]. > [1] > https://github.com/apache/beam/blob/master/sdks/python/apache_beam/io/gcp/bigquery.py > [2] > https://github.com/apache/beam/blob/master/sdks/python/apache_beam/io/iobase.py#L70 > [3] > https://github.com/apache/beam/blob/master/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/bigquery/BigQueryIO.java#L1189 -- This message was sent by Atlassian Jira (v8.3.4#803005)