ahmedabu98 commented on code in PR #28091:
URL: https://github.com/apache/beam/pull/28091#discussion_r1313022965
##########
sdks/python/apache_beam/io/gcp/bigquery_write_it_test.py:
##########
@@ -454,6 +454,56 @@ def test_big_query_write_insert_errors_reporting(self):
| 'ParseErrors' >> beam.Map(lambda err: (err[1], err[2])),
equal_to(bq_result_errors))
+
+ @pytest.mark.it_postcommit
+ def test_big_query_write_insert_not_found_errors(self):
+ """
+ Test that NotFound errors returned by beam.io.WriteToBigQuery
+ contain both the failed rows amd the reason for it failing.
Review Comment:
nit
```suggestion
contain both the failed rows and the reason for it failing.
```
##########
sdks/python/apache_beam/io/gcp/bigquery_tools.py:
##########
@@ -732,12 +732,23 @@ def _insert_all_rows(
except (ClientError, GoogleAPICallError) as e:
# e.code contains the numeric http status code.
service_call_metric.call(e.code)
- # Re-reise the exception so that we re-try appropriately.
- raise
- except HttpError as e:
+ # Package exception up with required fields
+ # Set reason to 'invalid' to consider these execptions as
RetryStrategy._NON_TRANSIENT_ERRORS
Review Comment:
> So if one error is retry-able, we could just raise the exception and let
Dataflow handle it
IIUC this is the current behavior already. When this exception is raised,
Dataflow sees it as a failed work item and retries it (batch 4 times, streaming
infinitely).
##########
sdks/python/apache_beam/io/gcp/bigquery_tools.py:
##########
@@ -732,12 +732,23 @@ def _insert_all_rows(
except (ClientError, GoogleAPICallError) as e:
# e.code contains the numeric http status code.
service_call_metric.call(e.code)
- # Re-reise the exception so that we re-try appropriately.
- raise
- except HttpError as e:
+ # Package exception up with required fields
+ # Set reason to 'invalid' to consider these execptions as
RetryStrategy._NON_TRANSIENT_ERRORS
+ error = {
+ 'message': e.message,
+ 'reason': 'invalid'
Review Comment:
I'd prefer a more granular approach than setting all these exceptions to
`invalid`. Are we sure these errors will always be non-transient? Perhaps we
can have an if-tree that sets an appropriate reason based on the error code. Or
maybe better to pass in the actual error reason and update
[`_NON_TRANSIENT_ERRORS`](https://github.com/apache/beam/blob/0b4302e5f95f2dc9b6658c13d5d1aa798cfba668/sdks/python/apache_beam/io/gcp/bigquery_tools.py#L1494)
to include more reasons.
For example the code `400` indicates an invalid/invalidQuery error so we can
say that's non-transient. But if we get something like a `500` or `503`
indicates a temporary error and BQ suggests retrying. More info in this [error
messages documentation](https://cloud.google.com/bigquery/docs/error-messages).
--
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.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]