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

Change regular expression to exclude double quote and newline #39991

Merged
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
2 changes: 1 addition & 1 deletion airflow/providers/google/cloud/hooks/dataflow.py
Original file line number Diff line number Diff line change
Expand Up @@ -71,7 +71,7 @@


JOB_ID_PATTERN = re.compile(
r"Submitted job: (?P<job_id_java>.*)|Created job with id: \[(?P<job_id_python>.*)\]"
r"Submitted job: (?P<job_id_java>[^\"\n]*)|Created job with id: \[(?P<job_id_python>[^\"\n]*)\]"
)

T = TypeVar("T", bound=Callable)
Expand Down
17 changes: 17 additions & 0 deletions tests/providers/google/cloud/hooks/test_dataflow.py
Original file line number Diff line number Diff line change
Expand Up @@ -1877,6 +1877,19 @@ def test_fetch_job_autoscaling_events_by_id(self, mock_fetch_responses):
> gcloud dataflow jobs --project=XXX cancel --region=europe-west3 {TEST_JOB_ID}
"""

CLOUD_COMPOSER_CLOUD_LOGGING_APACHE_BEAM_V_2_56_0_JAVA_SDK_LOG = f"""\
WARNING - {{"message":"org.apache.beam.runners.dataflow.DataflowRunner - Dataflow SDK version: 2.56.0",\
"severity":"INFO"}}
WARNING - {{"message":"org.apache.beam.runners.dataflow.DataflowRunner - To access the Dataflow monitoring\\
console, please navigate to https://console.cloud.google.com/dataflow/jobs/europe-west3/{TEST_JOB_ID}?project
\u003dXXXX","severity":"INFO"}}
WARNING - {{"message":"org.apache.beam.runners.dataflow.DataflowRunner - Submitted job: {TEST_JOB_ID}",\
"severity":"INFO"}}
WARNING - {{"message":"org.apache.beam.runners.dataflow.DataflowRunner - To cancel the job using the \
\u0027gcloud\u0027 tool, run:\n\u003e gcloud dataflow jobs --project\u003dXXX cancel --region\u003deurope-\
west3 {TEST_JOB_ID}","severity":"INFO"}}
"""

APACHE_BEAM_V_2_14_0_PYTHON_SDK_LOG = f""""\
INFO:root:Completed GCS upload to gs://test-dataflow-example/staging/start-python-job-local-5bcf3d71.\
1592286375.000962/apache_beam-2.14.0-cp37-cp37m-manylinux1_x86_64.whl in 0 seconds.
Expand Down Expand Up @@ -1926,6 +1939,10 @@ class TestDataflow:
[
pytest.param(APACHE_BEAM_V_2_14_0_JAVA_SDK_LOG, id="apache-beam-2.14.0-JDK"),
pytest.param(APACHE_BEAM_V_2_22_0_JAVA_SDK_LOG, id="apache-beam-2.22.0-JDK"),
pytest.param(
CLOUD_COMPOSER_CLOUD_LOGGING_APACHE_BEAM_V_2_56_0_JAVA_SDK_LOG,
id="cloud-composer-cloud-logging-apache-beam-2.56.0-JDK",
),
pytest.param(APACHE_BEAM_V_2_14_0_PYTHON_SDK_LOG, id="apache-beam-2.14.0-Python"),
pytest.param(APACHE_BEAM_V_2_22_0_PYTHON_SDK_LOG, id="apache-beam-2.22.0-Python"),
],
Expand Down