You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@airflow.apache.org by ka...@apache.org on 2022/06/27 20:29:57 UTC

[airflow] branch main updated: Add Core SQL Provider (#24476)

This is an automated email from the ASF dual-hosted git repository.

kaxilnaik pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/airflow.git


The following commit(s) were added to refs/heads/main by this push:
     new eb6f45e568 Add Core SQL Provider (#24476)
eb6f45e568 is described below

commit eb6f45e568fccdac30daf39f24e09d51c69a2cca
Author: Benji Lampel <be...@astronomer.io>
AuthorDate: Mon Jun 27 16:29:33 2022 -0400

    Add Core SQL Provider (#24476)
    
    Adds operators, tests, and new and updated docs for a Core SQL
    Provider. This provider is made in favor of adding these operators
    to the existing SQL operators in core Airflow. The new provider
    will allow for quicker development cycles.
    
    closes: #23874, #24422
    related: #23915
---
 .../airflow_providers_bug_report.yml               |   1 +
 CONTRIBUTING.rst                                   |   2 +-
 INSTALL                                            |   2 +-
 airflow/providers/core/__init__.py                 |  16 ++
 airflow/providers/core/sql/CHANGELOG.rst           |  26 ++
 airflow/providers/core/sql/__init__.py             |  16 ++
 .../providers/core/sql/example_dags/__init__.py    |  16 ++
 .../example_dags/example_sql_column_table_check.py |  77 +++++
 airflow/providers/core/sql/operators/__init__.py   |  16 ++
 airflow/providers/core/sql/operators/sql.py        | 315 +++++++++++++++++++++
 airflow/providers/core/sql/provider.yaml           |  41 +++
 docs/apache-airflow-providers-core-sql/commits.rst |  25 ++
 .../connections.rst                                |  32 +++
 docs/apache-airflow-providers-core-sql/index.rst   |  54 ++++
 .../installing-providers-from-sources.rst          |  18 ++
 .../operators.rst                                  | 112 ++++++++
 docs/apache-airflow/extra-packages-ref.rst         |  18 +-
 docs/integration-logos/core/sql/sql.png            | Bin 0 -> 19352 bytes
 docs/spelling_wordlist.txt                         |   2 +
 images/breeze/output-build-docs.svg                | 252 +++++++++--------
 images/breeze/output-commands-hash.txt             |   2 +-
 .../output-prepare-provider-documentation.svg      | 136 ++++-----
 images/breeze/output-prepare-provider-packages.svg | 136 ++++-----
 setup.py                                           |   1 +
 tests/providers/core/__init__.py                   |  16 ++
 tests/providers/core/sql/__init__.py               |  16 ++
 tests/providers/core/sql/operators/__init__.py     |  16 ++
 tests/providers/core/sql/operators/test_sql.py     | 114 ++++++++
 28 files changed, 1209 insertions(+), 269 deletions(-)

diff --git a/.github/ISSUE_TEMPLATE/airflow_providers_bug_report.yml b/.github/ISSUE_TEMPLATE/airflow_providers_bug_report.yml
index 70f1a27e7b..67be91e8cb 100644
--- a/.github/ISSUE_TEMPLATE/airflow_providers_bug_report.yml
+++ b/.github/ISSUE_TEMPLATE/airflow_providers_bug_report.yml
@@ -43,6 +43,7 @@ body:
         - celery
         - cloudant
         - cncf-kubernetes
+        - core-sql
         - databricks
         - datadog
         - dbt-cloud
diff --git a/CONTRIBUTING.rst b/CONTRIBUTING.rst
index a906c0eec2..4ec19f86ac 100644
--- a/CONTRIBUTING.rst
+++ b/CONTRIBUTING.rst
@@ -647,7 +647,7 @@ This is the full list of those extras:
 airbyte, alibaba, all, all_dbs, amazon, apache.atlas, apache.beam, apache.cassandra, apache.drill,
 apache.druid, apache.hdfs, apache.hive, apache.kylin, apache.livy, apache.pig, apache.pinot,
 apache.spark, apache.sqoop, apache.webhdfs, arangodb, asana, async, atlas, aws, azure, cassandra,
-celery, cgroups, cloudant, cncf.kubernetes, crypto, dask, databricks, datadog, dbt.cloud,
+celery, cgroups, cloudant, cncf.kubernetes, core.sql, crypto, dask, databricks, datadog, dbt.cloud,
 deprecated_api, devel, devel_all, devel_ci, devel_hadoop, dingding, discord, doc, docker, druid,
 elasticsearch, exasol, facebook, ftp, gcp, gcp_api, github, github_enterprise, google, google_auth,
 grpc, hashicorp, hdfs, hive, http, imap, influxdb, jdbc, jenkins, jira, kerberos, kubernetes, ldap,
diff --git a/INSTALL b/INSTALL
index ba2f550047..9694ba1f5b 100644
--- a/INSTALL
+++ b/INSTALL
@@ -97,7 +97,7 @@ The list of available extras:
 airbyte, alibaba, all, all_dbs, amazon, apache.atlas, apache.beam, apache.cassandra, apache.drill,
 apache.druid, apache.hdfs, apache.hive, apache.kylin, apache.livy, apache.pig, apache.pinot,
 apache.spark, apache.sqoop, apache.webhdfs, arangodb, asana, async, atlas, aws, azure, cassandra,
-celery, cgroups, cloudant, cncf.kubernetes, crypto, dask, databricks, datadog, dbt.cloud,
+celery, cgroups, cloudant, cncf.kubernetes, core.sql, crypto, dask, databricks, datadog, dbt.cloud,
 deprecated_api, devel, devel_all, devel_ci, devel_hadoop, dingding, discord, doc, docker, druid,
 elasticsearch, exasol, facebook, ftp, gcp, gcp_api, github, github_enterprise, google, google_auth,
 grpc, hashicorp, hdfs, hive, http, imap, influxdb, jdbc, jenkins, jira, kerberos, kubernetes, ldap,
diff --git a/airflow/providers/core/__init__.py b/airflow/providers/core/__init__.py
new file mode 100644
index 0000000000..13a83393a9
--- /dev/null
+++ b/airflow/providers/core/__init__.py
@@ -0,0 +1,16 @@
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
diff --git a/airflow/providers/core/sql/CHANGELOG.rst b/airflow/providers/core/sql/CHANGELOG.rst
new file mode 100644
index 0000000000..a5b4657ed3
--- /dev/null
+++ b/airflow/providers/core/sql/CHANGELOG.rst
@@ -0,0 +1,26 @@
+ .. Licensed to the Apache Software Foundation (ASF) under one
+    or more contributor license agreements.  See the NOTICE file
+    distributed with this work for additional information
+    regarding copyright ownership.  The ASF licenses this file
+    to you under the Apache License, Version 2.0 (the
+    "License"); you may not use this file except in compliance
+    with the License.  You may obtain a copy of the License at
+
+ ..   http://www.apache.org/licenses/LICENSE-2.0
+
+ .. Unless required by applicable law or agreed to in writing,
+    software distributed under the License is distributed on an
+    "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+    KIND, either express or implied.  See the License for the
+    specific language governing permissions and limitations
+    under the License.
+
+
+Changelog
+---------
+
+1.0.0
+.....
+
+Initial version of the provider.
+Adds SQLColumnCheckOperator and SQLTableCheckOperator.
diff --git a/airflow/providers/core/sql/__init__.py b/airflow/providers/core/sql/__init__.py
new file mode 100644
index 0000000000..13a83393a9
--- /dev/null
+++ b/airflow/providers/core/sql/__init__.py
@@ -0,0 +1,16 @@
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
diff --git a/airflow/providers/core/sql/example_dags/__init__.py b/airflow/providers/core/sql/example_dags/__init__.py
new file mode 100644
index 0000000000..13a83393a9
--- /dev/null
+++ b/airflow/providers/core/sql/example_dags/__init__.py
@@ -0,0 +1,16 @@
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
diff --git a/airflow/providers/core/sql/example_dags/example_sql_column_table_check.py b/airflow/providers/core/sql/example_dags/example_sql_column_table_check.py
new file mode 100644
index 0000000000..e83f021765
--- /dev/null
+++ b/airflow/providers/core/sql/example_dags/example_sql_column_table_check.py
@@ -0,0 +1,77 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from airflow import DAG
+from airflow.providers.core.sql.operators.sql import SQLColumnCheckOperator, SQLTableCheckOperator
+from airflow.utils.dates import datetime
+
+AIRFLOW_DB_METADATA_TABLE = "ab_role"
+connection_args = {
+    "conn_id": "airflow_db",
+    "conn_type": "Postgres",
+    "host": "postgres",
+    "schema": "postgres",
+    "login": "postgres",
+    "password": "postgres",
+    "port": 5432,
+}
+
+with DAG(
+    "example_sql_column_table_check",
+    description="Example DAG for SQLColumnCheckOperator and SQLTableCheckOperator.",
+    default_args=connection_args,
+    start_date=datetime(2021, 1, 1),
+    schedule_interval=None,
+    catchup=False,
+) as dag:
+    """
+    ### Example SQL Column and Table Check DAG
+
+    Runs the SQLColumnCheckOperator and SQLTableCheckOperator against the Airflow metadata DB.
+    """
+
+    # [START howto_operator_sql_column_check]
+    column_check = SQLColumnCheckOperator(
+        task_id="column_check",
+        table=AIRFLOW_DB_METADATA_TABLE,
+        column_mapping={
+            "id": {
+                "null_check": {
+                    "equal_to": 0,
+                    "tolerance": 0,
+                },
+                "distinct_check": {
+                    "equal_to": 1,
+                },
+            }
+        },
+    )
+    # [END howto_operator_sql_column_check]
+
+    # [START howto_operator_sql_table_check]
+    row_count_check = SQLTableCheckOperator(
+        task_id="row_count_check",
+        table=AIRFLOW_DB_METADATA_TABLE,
+        checks={
+            "row_count_check": {
+                "check_statement": "COUNT(*) = 1",
+            }
+        },
+    )
+    # [END howto_operator_sql_table_check]
+
+    column_check >> row_count_check
diff --git a/airflow/providers/core/sql/operators/__init__.py b/airflow/providers/core/sql/operators/__init__.py
new file mode 100644
index 0000000000..13a83393a9
--- /dev/null
+++ b/airflow/providers/core/sql/operators/__init__.py
@@ -0,0 +1,16 @@
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
diff --git a/airflow/providers/core/sql/operators/sql.py b/airflow/providers/core/sql/operators/sql.py
new file mode 100644
index 0000000000..1d9945f403
--- /dev/null
+++ b/airflow/providers/core/sql/operators/sql.py
@@ -0,0 +1,315 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Any, Dict, Optional, Union
+
+from airflow.exceptions import AirflowException
+from airflow.operators.sql import BaseSQLOperator
+
+
+def parse_boolean(val: str) -> Union[str, bool]:
+    """Try to parse a string into boolean.
+
+    Raises ValueError if the input is not a valid true- or false-like string value.
+    """
+    val = val.lower()
+    if val in ('y', 'yes', 't', 'true', 'on', '1'):
+        return True
+    if val in ('n', 'no', 'f', 'false', 'off', '0'):
+        return False
+    raise ValueError(f"{val!r} is not a boolean-like string value")
+
+
+def _get_failed_tests(checks):
+    return [
+        f"\tCheck: {check},\n\tCheck Values: {check_values}\n"
+        for check, check_values in checks.items()
+        if not check_values["success"]
+    ]
+
+
+class SQLColumnCheckOperator(BaseSQLOperator):
+    """
+    Performs one or more of the templated checks in the column_checks dictionary.
+    Checks are performed on a per-column basis specified by the column_mapping.
+    Each check can take one or more of the following options:
+    - equal_to: an exact value to equal, cannot be used with other comparison options
+    - greater_than: value that result should be strictly greater than
+    - less_than: value that results should be strictly less than
+    - geq_to: value that results should be greater than or equal to
+    - leq_to: value that results should be less than or equal to
+    - tolerance: the percentage that the result may be off from the expected value
+
+    :param table: the table to run checks on
+    :param column_mapping: the dictionary of columns and their associated checks, e.g.
+
+    .. code-block:: python
+
+        {
+            "col_name": {
+                "null_check": {
+                    "equal_to": 0,
+                },
+                "min": {
+                    "greater_than": 5,
+                    "leq_to": 10,
+                    "tolerance": 0.2,
+                },
+                "max": {"less_than": 1000, "geq_to": 10, "tolerance": 0.01},
+            }
+        }
+
+    :param conn_id: the connection ID used to connect to the database
+    :param database: name of database which overwrite the defined one in connection
+
+    .. seealso::
+        For more information on how to use this operator, take a look at the guide:
+        :ref:`howto/operator:SQLColumnCheckOperator`
+    """
+
+    column_checks = {
+        "null_check": "SUM(CASE WHEN column IS NULL THEN 1 ELSE 0 END) AS column_null_check",
+        "distinct_check": "COUNT(DISTINCT(column)) AS column_distinct_check",
+        "unique_check": "COUNT(column) - COUNT(DISTINCT(column)) AS column_unique_check",
+        "min": "MIN(column) AS column_min",
+        "max": "MAX(column) AS column_max",
+    }
+
+    def __init__(
+        self,
+        *,
+        table: str,
+        column_mapping: Dict[str, Dict[str, Any]],
+        conn_id: Optional[str] = None,
+        database: Optional[str] = None,
+        **kwargs,
+    ):
+        super().__init__(conn_id=conn_id, database=database, **kwargs)
+        for checks in column_mapping.values():
+            for check, check_values in checks.items():
+                self._column_mapping_validation(check, check_values)
+
+        self.table = table
+        self.column_mapping = column_mapping
+        # OpenLineage needs a valid SQL query with the input/output table(s) to parse
+        self.sql = f"SELECT * FROM {self.table};"
+
+    def execute(self, context=None):
+        hook = self.get_db_hook()
+        failed_tests = []
+        for column in self.column_mapping:
+            checks = [*self.column_mapping[column]]
+            checks_sql = ",".join([self.column_checks[check].replace("column", column) for check in checks])
+
+            self.sql = f"SELECT {checks_sql} FROM {self.table};"
+            records = hook.get_first(self.sql)
+
+            if not records:
+                raise AirflowException(f"The following query returned zero rows: {self.sql}")
+
+            self.log.info(f"Record: {records}")
+
+            for idx, result in enumerate(records):
+                tolerance = self.column_mapping[column][checks[idx]].get("tolerance")
+
+                self.column_mapping[column][checks[idx]]["result"] = result
+                self.column_mapping[column][checks[idx]]["success"] = self._get_match(
+                    self.column_mapping[column][checks[idx]], result, tolerance
+                )
+
+            failed_tests.extend(_get_failed_tests(self.column_mapping[column]))
+        if failed_tests:
+            raise AirflowException(
+                f"Test failed.\nQuery:\n{self.sql}\nResults:\n{records!s}\n"
+                "The following tests have failed:"
+                f"\n{''.join(failed_tests)}"
+            )
+
+        self.log.info("All tests have passed")
+
+    def _get_match(self, check_values, record, tolerance=None) -> bool:
+        match_boolean = True
+        if "geq_to" in check_values:
+            if tolerance is not None:
+                match_boolean = record >= check_values["geq_to"] * (1 - tolerance)
+            else:
+                match_boolean = record >= check_values["geq_to"]
+        elif "greater_than" in check_values:
+            if tolerance is not None:
+                match_boolean = record > check_values["greater_than"] * (1 - tolerance)
+            else:
+                match_boolean = record > check_values["greater_than"]
+        if "leq_to" in check_values:
+            if tolerance is not None:
+                match_boolean = record <= check_values["leq_to"] * (1 + tolerance) and match_boolean
+            else:
+                match_boolean = record <= check_values["leq_to"] and match_boolean
+        elif "less_than" in check_values:
+            if tolerance is not None:
+                match_boolean = record < check_values["less_than"] * (1 + tolerance) and match_boolean
+            else:
+                match_boolean = record < check_values["less_than"] and match_boolean
+        if "equal_to" in check_values:
+            if tolerance is not None:
+                match_boolean = (
+                    check_values["equal_to"] * (1 - tolerance)
+                    <= record
+                    <= check_values["equal_to"] * (1 + tolerance)
+                ) and match_boolean
+            else:
+                match_boolean = record == check_values["equal_to"] and match_boolean
+        return match_boolean
+
+    def _column_mapping_validation(self, check, check_values):
+        if check not in self.column_checks:
+            raise AirflowException(f"Invalid column check: {check}.")
+        if (
+            "greater_than" not in check_values
+            and "geq_to" not in check_values
+            and "less_than" not in check_values
+            and "leq_to" not in check_values
+            and "equal_to" not in check_values
+        ):
+            raise ValueError(
+                "Please provide one or more of: less_than, leq_to, "
+                "greater_than, geq_to, or equal_to in the check's dict."
+            )
+
+        if "greater_than" in check_values and "less_than" in check_values:
+            if check_values["greater_than"] >= check_values["less_than"]:
+                raise ValueError(
+                    "greater_than should be strictly less than "
+                    "less_than. Use geq_to or leq_to for "
+                    "overlapping equality."
+                )
+
+        if "greater_than" in check_values and "leq_to" in check_values:
+            if check_values["greater_than"] >= check_values["leq_to"]:
+                raise ValueError(
+                    "greater_than must be strictly less than leq_to. "
+                    "Use geq_to with leq_to for overlapping equality."
+                )
+
+        if "geq_to" in check_values and "less_than" in check_values:
+            if check_values["geq_to"] >= check_values["less_than"]:
+                raise ValueError(
+                    "geq_to should be strictly less than less_than. "
+                    "Use leq_to with geq_to for overlapping equality."
+                )
+
+        if "geq_to" in check_values and "leq_to" in check_values:
+            if check_values["geq_to"] > check_values["leq_to"]:
+                raise ValueError("geq_to should be less than or equal to leq_to.")
+
+        if "greater_than" in check_values and "geq_to" in check_values:
+            raise ValueError("Only supply one of greater_than or geq_to.")
+
+        if "less_than" in check_values and "leq_to" in check_values:
+            raise ValueError("Only supply one of less_than or leq_to.")
+
+        if (
+            "greater_than" in check_values
+            or "geq_to" in check_values
+            or "less_than" in check_values
+            or "leq_to" in check_values
+        ) and "equal_to" in check_values:
+            raise ValueError(
+                "equal_to cannot be passed with a greater or less than "
+                "function. To specify 'greater than or equal to' or "
+                "'less than or equal to', use geq_to or leq_to."
+            )
+
+
+class SQLTableCheckOperator(BaseSQLOperator):
+    """
+    Performs one or more of the checks provided in the checks dictionary.
+    Checks should be written to return a boolean result.
+
+    :param table: the table to run checks on
+    :param checks: the dictionary of checks, e.g.:
+
+    .. code-block:: python
+
+        {
+            "row_count_check": {"check_statement": "COUNT(*) = 1000"},
+            "column_sum_check": {"check_statement": "col_a + col_b < col_c"},
+        }
+
+    :param conn_id: the connection ID used to connect to the database
+    :param database: name of database which overwrite the defined one in connection
+
+    .. seealso::
+        For more information on how to use this operator, take a look at the guide:
+        :ref:`howto/operator:SQLTableCheckOperator`
+    """
+
+    sql_check_template = "CASE WHEN check_statement THEN 1 ELSE 0 END AS check_name"
+    sql_min_template = "MIN(check_name)"
+
+    def __init__(
+        self,
+        *,
+        table: str,
+        checks: Dict[str, Dict[str, Any]],
+        conn_id: Optional[str] = None,
+        database: Optional[str] = None,
+        **kwargs,
+    ):
+        super().__init__(conn_id=conn_id, database=database, **kwargs)
+
+        self.table = table
+        self.checks = checks
+        # OpenLineage needs a valid SQL query with the input/output table(s) to parse
+        self.sql = f"SELECT * FROM {self.table};"
+
+    def execute(self, context=None):
+        hook = self.get_db_hook()
+
+        check_names = [*self.checks]
+        check_mins_sql = ",".join(
+            self.sql_min_template.replace("check_name", check_name) for check_name in check_names
+        )
+        checks_sql = ",".join(
+            [
+                self.sql_check_template.replace("check_statement", value["check_statement"]).replace(
+                    "check_name", check_name
+                )
+                for check_name, value in self.checks.items()
+            ]
+        )
+
+        self.sql = f"SELECT {check_mins_sql} FROM (SELECT {checks_sql} FROM {self.table});"
+        records = hook.get_first(self.sql)
+
+        if not records:
+            raise AirflowException(f"The following query returned zero rows: {self.sql}")
+
+        self.log.info(f"Record: {records}")
+
+        for check in self.checks.keys():
+            for result in records:
+                self.checks[check]["success"] = parse_boolean(str(result))
+
+        failed_tests = _get_failed_tests(self.checks)
+        if failed_tests:
+            raise AirflowException(
+                f"Test failed.\nQuery:\n{self.sql}\nResults:\n{records!s}\n"
+                "The following tests have failed:"
+                f"\n{', '.join(failed_tests)}"
+            )
+
+        self.log.info("All tests have passed")
diff --git a/airflow/providers/core/sql/provider.yaml b/airflow/providers/core/sql/provider.yaml
new file mode 100644
index 0000000000..64f6d9746c
--- /dev/null
+++ b/airflow/providers/core/sql/provider.yaml
@@ -0,0 +1,41 @@
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+
+---
+package-name: apache-airflow-providers-core-sql
+name: Core SQL
+description: |
+    `Core SQL Provider <https://en.wikipedia.org/wiki/SQL>`__
+
+versions:
+  - 1.0.0
+
+additional-dependencies:
+  - apache-airflow>=2.2.0
+
+integrations:
+  - integration-name: Core SQL
+    external-doc-url: https://en.wikipedia.org/wiki/SQL
+    how-to-guide:
+      - /docs/apache-airflow-providers-core-sql/operators.rst
+    logo: /integration-logos/core/sql/sql.png
+    tags: [software]
+
+operators:
+  - integration-name: Core SQL
+    python-modules:
+      - airflow.providers.core.sql.operators.sql
diff --git a/docs/apache-airflow-providers-core-sql/commits.rst b/docs/apache-airflow-providers-core-sql/commits.rst
new file mode 100644
index 0000000000..8292a11950
--- /dev/null
+++ b/docs/apache-airflow-providers-core-sql/commits.rst
@@ -0,0 +1,25 @@
+ .. Licensed to the Apache Software Foundation (ASF) under one
+    or more contributor license agreements.  See the NOTICE file
+    distributed with this work for additional information
+    regarding copyright ownership.  The ASF licenses this file
+    to you under the Apache License, Version 2.0 (the
+    "License"); you may not use this file except in compliance
+    with the License.  You may obtain a copy of the License at
+
+ ..   http://www.apache.org/licenses/LICENSE-2.0
+
+ .. Unless required by applicable law or agreed to in writing,
+    software distributed under the License is distributed on an
+    "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+    KIND, either express or implied.  See the License for the
+    specific language governing permissions and limitations
+    under the License.
+
+Package apache-airflow-providers-core-sql
+------------------------------------------
+
+`Core SQL Provider <https://en.wikipedia.org/wiki/SQL>`__
+
+
+This is detailed commit list of changes for versions provider package: ``core.sql``.
+For high-level changelog, see :doc:`package information including changelog <index>`.
diff --git a/docs/apache-airflow-providers-core-sql/connections.rst b/docs/apache-airflow-providers-core-sql/connections.rst
new file mode 100644
index 0000000000..ad06ed4b8e
--- /dev/null
+++ b/docs/apache-airflow-providers-core-sql/connections.rst
@@ -0,0 +1,32 @@
+ .. Licensed to the Apache Software Foundation (ASF) under one
+    or more contributor license agreements.  See the NOTICE file
+    distributed with this work for additional information
+    regarding copyright ownership.  The ASF licenses this file
+    to you under the Apache License, Version 2.0 (the
+    "License"); you may not use this file except in compliance
+    with the License.  You may obtain a copy of the License at
+
+ ..   http://www.apache.org/licenses/LICENSE-2.0
+
+ .. Unless required by applicable law or agreed to in writing,
+    software distributed under the License is distributed on an
+    "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+    KIND, either express or implied.  See the License for the
+    specific language governing permissions and limitations
+    under the License.
+
+.. _howto/connection:sql:
+
+Connecting to a SQL DB
+=======================
+
+The SQL Provider package operators allow access to various SQL-like databases. For
+databases that can be connected to with a DBApi Hook directly, simply passing the
+connection ID with these operators is sufficient. For other connections with more
+complicated hooks, the additional parameters can be passed as key word args to the
+operators.
+
+Default Connection ID
+~~~~~~~~~~~~~~~~~~~~~
+
+SQL Operators under this provider do not default to any connection ID.
diff --git a/docs/apache-airflow-providers-core-sql/index.rst b/docs/apache-airflow-providers-core-sql/index.rst
new file mode 100644
index 0000000000..abc6186b68
--- /dev/null
+++ b/docs/apache-airflow-providers-core-sql/index.rst
@@ -0,0 +1,54 @@
+ .. Licensed to the Apache Software Foundation (ASF) under one
+    or more contributor license agreements.  See the NOTICE file
+    distributed with this work for additional information
+    regarding copyright ownership.  The ASF licenses this file
+    to you under the Apache License, Version 2.0 (the
+    "License"); you may not use this file except in compliance
+    with the License.  You may obtain a copy of the License at
+
+ ..   http://www.apache.org/licenses/LICENSE-2.0
+
+ .. Unless required by applicable law or agreed to in writing,
+    software distributed under the License is distributed on an
+    "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+    KIND, either express or implied.  See the License for the
+    specific language governing permissions and limitations
+    under the License.
+
+``apache-airflow-providers-core-sql``
+============================================
+
+Content
+-------
+
+.. toctree::
+    :maxdepth: 1
+    :caption: Guides
+
+    Connecting to SQL Databases <connections>
+    Operators <operators>
+
+.. toctree::
+    :maxdepth: 1
+    :caption: References
+
+    Python API <_api/airflow/providers/core/sql/index>
+
+.. toctree::
+    :maxdepth: 1
+    :caption: Resources
+
+    Example DAGs <https://github.com/apache/airflow/tree/main/airflow/providers/core/sql/example_dags>
+    PyPI Repository <https://pypi.org/project/apache-airflow-providers-core-sql/>
+    Installing from sources <installing-providers-from-sources>
+
+.. THE REMAINDER OF THE FILE IS AUTOMATICALLY GENERATED. IT WILL BE OVERWRITTEN AT RELEASE TIME!
+.. toctree::
+    :maxdepth: 1
+    :caption: Commits
+
+    Detailed list of commits <commits>
+
+
+Package apache-airflow-providers-core-sql
+------------------------------------------------------
diff --git a/docs/apache-airflow-providers-core-sql/installing-providers-from-sources.rst b/docs/apache-airflow-providers-core-sql/installing-providers-from-sources.rst
new file mode 100644
index 0000000000..1c90205d15
--- /dev/null
+++ b/docs/apache-airflow-providers-core-sql/installing-providers-from-sources.rst
@@ -0,0 +1,18 @@
+ .. Licensed to the Apache Software Foundation (ASF) under one
+    or more contributor license agreements.  See the NOTICE file
+    distributed with this work for additional information
+    regarding copyright ownership.  The ASF licenses this file
+    to you under the Apache License, Version 2.0 (the
+    "License"); you may not use this file except in compliance
+    with the License.  You may obtain a copy of the License at
+
+ ..   http://www.apache.org/licenses/LICENSE-2.0
+
+ .. Unless required by applicable law or agreed to in writing,
+    software distributed under the License is distributed on an
+    "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+    KIND, either express or implied.  See the License for the
+    specific language governing permissions and limitations
+    under the License.
+
+.. include:: ../installing-providers-from-sources.rst
diff --git a/docs/apache-airflow-providers-core-sql/operators.rst b/docs/apache-airflow-providers-core-sql/operators.rst
new file mode 100644
index 0000000000..c2b04767d4
--- /dev/null
+++ b/docs/apache-airflow-providers-core-sql/operators.rst
@@ -0,0 +1,112 @@
+ .. Licensed to the Apache Software Foundation (ASF) under one
+    or more contributor license agreements.  See the NOTICE file
+    distributed with this work for additional information
+    regarding copyright ownership.  The ASF licenses this file
+    to you under the Apache License, Version 2.0 (the
+    "License"); you may not use this file except in compliance
+    with the License.  You may obtain a copy of the License at
+
+ ..   http://www.apache.org/licenses/LICENSE-2.0
+
+ .. Unless required by applicable law or agreed to in writing,
+    software distributed under the License is distributed on an
+    "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+    KIND, either express or implied.  See the License for the
+    specific language governing permissions and limitations
+    under the License.
+
+SQL Operators
+===================
+
+These operators perform various queries against a SQL database, including
+column- and table-level data quality checks.
+
+.. _howto/operator:SQLColumnCheckOperator:
+
+Check SQL Table Columns
+~~~~~~~~~~~~~~~~~~~~~~~
+
+Use the :class:`~airflow.providers.core.sql.operators.sql.SQLColumnCheckOperator` to run data quality
+checks against columns of a given table. As well as a connection ID and table, a column_mapping
+describing the relationship between columns and tests to run must be supplied. An example column
+mapping is a set of three nested dictionaries and looks like:
+
+.. code-block:: python
+
+        column_mapping = {
+            "col_name": {
+                "null_check": {
+                    "equal_to": 0,
+                },
+                "min": {
+                    "greater_than": 5,
+                    "leq_to": 10,
+                    "tolerance": 0.2,
+                },
+                "max": {"less_than": 1000, "geq_to": 10, "tolerance": 0.01},
+            }
+        }
+
+Where col_name is the name of the column to run checks on, and each entry in its dictionary is a check.
+The valid checks are:
+- null_check: checks the number of NULL values in the column
+- distinct_check: checks the COUNT of values in the column that are distinct
+- unique_check: checks the number of distinct values in a column against the number of rows
+- min: checks the minimum value in the column
+- max: checks the maximum value in the column
+
+Each entry in the check's dictionary is either a condition for success of the check or the tolerance. The
+conditions for success are:
+- greater_than
+- geq_to
+- less_than
+- leq_to
+- equal_to
+
+When specifying conditions, equal_to is not compatible with other conditions. Both a lower- and an upper-
+bound condition may be specified in the same check. The tolerance is a percentage that the result may
+be out of bounds but still considered successful.
+
+
+
+The below example demonstrates how to instantiate the SQLColumnCheckOperator task.
+
+.. exampleinclude:: /../../airflow/providers/core/sql/example_dags/example_sql_column_table_check.py
+    :language: python
+    :dedent: 4
+    :start-after: [START howto_operator_sql_column_check]
+    :end-before: [END howto_operator_sql_column_check]
+
+.. _howto/operator:SQLTableCheckOperator:
+
+Check SQL Table Values
+~~~~~~~~~~~~~~~~~~~~~~~
+
+Use the :class:`~airflow.providers.core.sql.operators.sql.SQLTableCheckOperator` to run data quality
+checks against a given table. As well as a connection ID and table, a checks dictionary
+describing the relationship between the table and tests to run must be supplied. An example
+checks argument is a set of two nested dictionaries and looks like:
+
+.. code-block:: python
+
+        checks = (
+            {
+                "row_count_check": {
+                    "check_statement": "COUNT(*) = 1000",
+                },
+                "column_sum_check": {"check_statement": "col_a + col_b < col_c"},
+            },
+        )
+
+The first set of keys are the check names, which are referenced in the templated query the operator builds.
+The dictionary key under the check name must be check_statement, with the value a SQL statement that
+resolves to a boolean (this can be any string or int that resolves to a boolean in
+airflow.operators.sql.parse_boolean).
+
+The below example demonstrates how to instantiate the SQLTableCheckOperator task.
+
+.. exampleinclude:: /../../airflow/providers/core/sql/example_dags/example_sql_column_table_check.py
+    :language: python
+    :dedent: 4
+    :start-after: [START howto_operator_sql_table_check]
+    :end-before: [END howto_operator_sql_table_check]
diff --git a/docs/apache-airflow/extra-packages-ref.rst b/docs/apache-airflow/extra-packages-ref.rst
index cefa4a67bf..188b9a02f9 100644
--- a/docs/apache-airflow/extra-packages-ref.rst
+++ b/docs/apache-airflow/extra-packages-ref.rst
@@ -33,7 +33,7 @@ pre-installed when Airflow is installed.
 Core Airflow extras
 -------------------
 
-Those are core airflow extras that extend capabilities of core Airflow. They usually do not install provider
+These are core airflow extras that extend capabilities of core Airflow. They usually do not install provider
 packages (with the exception of ``celery`` and ``cncf.kubernetes`` extras), they just install necessary
 python dependencies for the provided package.
 
@@ -79,7 +79,7 @@ python dependencies for the provided package.
 Providers extras
 ----------------
 
-Those providers extras are simply convenience extras to install provider packages so that you can install the providers with simple command - including
+These providers extras are simply convenience extras to install provider packages so that you can install the providers with simple command - including
 provider package and necessary dependencies in single command, which allows PIP to resolve any conflicting dependencies. This is extremely useful
 for first time installation where you want to repeatably install version of dependencies which are 'valid' for both airflow and providers installed.
 
@@ -105,7 +105,7 @@ upgrade those providers manually if you want to use latest versions of the provi
 Apache Software extras
 ======================
 
-Those are extras that add dependencies needed for integration with other Apache projects (note that ``apache.atlas`` and
+These are extras that add dependencies needed for integration with other Apache projects (note that ``apache.atlas`` and
 ``apache.webhdfs`` do not have their own providers - they only install additional libraries that can be used in
 custom bash/python providers).
 
@@ -145,7 +145,7 @@ custom bash/python providers).
 External Services extras
 ========================
 
-Those are extras that add dependencies needed for integration with external services - either cloud based or on-premises.
+These are extras that add dependencies needed for integration with external services - either cloud based or on-premises.
 
 +---------------------+-----------------------------------------------------+-----------------------------------------------------+
 | extra               | install command                                     | enables                                             |
@@ -213,7 +213,7 @@ Those are extras that add dependencies needed for integration with external serv
 Locally installed software extras
 =================================
 
-Those are extras that add dependencies needed for integration with other software packages installed usually as part of the deployment of Airflow.
+These are extras that add dependencies needed for integration with other software packages installed usually as part of the deployment of Airflow.
 
 +---------------------+-----------------------------------------------------+-------------------------------------------+
 | extra               | install command                                     | enables                                   |
@@ -263,11 +263,13 @@ Those are extras that add dependencies needed for integration with other softwar
 Other extras
 ============
 
-Those are extras that provide support for integration with external systems via some - usually - standard protocols.
+These are extras that provide support for integration with external systems via some - usually - standard protocols.
 
 +---------------------+-----------------------------------------------------+--------------------------------------+--------------+
 | extra               | install command                                     | enables                              | Preinstalled |
 +=====================+=====================================================+======================================+==============+
+| core.sql            | ``pip install 'apache-airflow[core.sql]'``          | Core SQL Operators                   |              |
++---------------------+-----------------------------------------------------+--------------------------------------+--------------+
 | ftp                 | ``pip install 'apache-airflow[ftp]'``               | FTP hooks and operators              |      *       |
 +---------------------+-----------------------------------------------------+--------------------------------------+--------------+
 | grpc                | ``pip install 'apache-airflow[grpc]'``              | Grpc hooks and operators             |              |
@@ -294,7 +296,7 @@ Those are extras that provide support for integration with external systems via
 Bundle extras
 -------------
 
-Those are extras that install one ore more extras as a bundle. Note that those extras should only be used for "development" version
+These are extras that install one ore more extras as a bundle. Note that these extras should only be used for "development" version
 of Airflow - i.e. when Airflow is installed from sources. Because of the way how bundle extras are constructed they might not
 work when airflow is installed from 'PyPI`.
 
@@ -332,7 +334,7 @@ This is the extra that is needed to generated documentation for Airflow. This is
 Deprecated 1.10 extras
 ----------------------
 
-Those are the extras that have been deprecated in 2.0 and will be removed in Airflow 3.0.0. They were
+These are the extras that have been deprecated in 2.0 and will be removed in Airflow 3.0.0. They were
 all replaced by new extras, which have naming consistent with the names of provider packages.
 
 The ``crypto`` extra is not needed any more, because all crypto dependencies are part of airflow package,
diff --git a/docs/integration-logos/core/sql/sql.png b/docs/integration-logos/core/sql/sql.png
new file mode 100644
index 0000000000..29a2685d8b
Binary files /dev/null and b/docs/integration-logos/core/sql/sql.png differ
diff --git a/docs/spelling_wordlist.txt b/docs/spelling_wordlist.txt
index d91c742ab1..44362b9b93 100644
--- a/docs/spelling_wordlist.txt
+++ b/docs/spelling_wordlist.txt
@@ -859,6 +859,7 @@ gcpcloudsql
 gcs
 gdbm
 generateUploadUrl
+geq
 getattr
 getfqdn
 getframe
@@ -1016,6 +1017,7 @@ latencies
 latin
 ldap
 ldaps
+leq
 leveldb
 libs
 libz
diff --git a/images/breeze/output-build-docs.svg b/images/breeze/output-build-docs.svg
index 928f8f898b..069d64d07e 100644
--- a/images/breeze/output-build-docs.svg
+++ b/images/breeze/output-build-docs.svg
@@ -1,4 +1,4 @@
-<svg class="rich-terminal" viewBox="0 0 1482 1392.0" xmlns="http://www.w3.org/2000/svg">
+<svg class="rich-terminal" viewBox="0 0 1482 1416.3999999999999" xmlns="http://www.w3.org/2000/svg">
     <!-- Generated with Rich https://www.textualize.io -->
     <style>
 
@@ -19,261 +19,265 @@
         font-weight: 700;
     }
 
-    .terminal-1803387802-matrix {
+    .terminal-3189088113-matrix {
         font-family: Fira Code, monospace;
         font-size: 20px;
         line-height: 24.4px;
         font-variant-east-asian: full-width;
     }
 
-    .terminal-1803387802-title {
+    .terminal-3189088113-title {
         font-size: 18px;
         font-weight: bold;
         font-family: arial;
     }
 
-    .terminal-1803387802-r1 { fill: #c5c8c6;font-weight: bold }
-.terminal-1803387802-r2 { fill: #c5c8c6 }
-.terminal-1803387802-r3 { fill: #d0b344;font-weight: bold }
-.terminal-1803387802-r4 { fill: #868887 }
-.terminal-1803387802-r5 { fill: #68a0b3;font-weight: bold }
-.terminal-1803387802-r6 { fill: #98a84b;font-weight: bold }
-.terminal-1803387802-r7 { fill: #8d7b39 }
+    .terminal-3189088113-r1 { fill: #c5c8c6;font-weight: bold }
+.terminal-3189088113-r2 { fill: #c5c8c6 }
+.terminal-3189088113-r3 { fill: #d0b344;font-weight: bold }
+.terminal-3189088113-r4 { fill: #868887 }
+.terminal-3189088113-r5 { fill: #68a0b3;font-weight: bold }
+.terminal-3189088113-r6 { fill: #98a84b;font-weight: bold }
+.terminal-3189088113-r7 { fill: #8d7b39 }
     </style>
 
     <defs>
-    <clipPath id="terminal-1803387802-clip-terminal">
-      <rect x="0" y="0" width="1463.0" height="1341.0" />
+    <clipPath id="terminal-3189088113-clip-terminal">
+      <rect x="0" y="0" width="1463.0" height="1365.3999999999999" />
     </clipPath>
-    <clipPath id="terminal-1803387802-line-0">
+    <clipPath id="terminal-3189088113-line-0">
     <rect x="0" y="1.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-1">
+<clipPath id="terminal-3189088113-line-1">
     <rect x="0" y="25.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-2">
+<clipPath id="terminal-3189088113-line-2">
     <rect x="0" y="50.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-3">
+<clipPath id="terminal-3189088113-line-3">
     <rect x="0" y="74.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-4">
+<clipPath id="terminal-3189088113-line-4">
     <rect x="0" y="99.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-5">
+<clipPath id="terminal-3189088113-line-5">
     <rect x="0" y="123.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-6">
+<clipPath id="terminal-3189088113-line-6">
     <rect x="0" y="147.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-7">
+<clipPath id="terminal-3189088113-line-7">
     <rect x="0" y="172.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-8">
+<clipPath id="terminal-3189088113-line-8">
     <rect x="0" y="196.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-9">
+<clipPath id="terminal-3189088113-line-9">
     <rect x="0" y="221.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-10">
+<clipPath id="terminal-3189088113-line-10">
     <rect x="0" y="245.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-11">
+<clipPath id="terminal-3189088113-line-11">
     <rect x="0" y="269.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-12">
+<clipPath id="terminal-3189088113-line-12">
     <rect x="0" y="294.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-13">
+<clipPath id="terminal-3189088113-line-13">
     <rect x="0" y="318.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-14">
+<clipPath id="terminal-3189088113-line-14">
     <rect x="0" y="343.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-15">
+<clipPath id="terminal-3189088113-line-15">
     <rect x="0" y="367.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-16">
+<clipPath id="terminal-3189088113-line-16">
     <rect x="0" y="391.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-17">
+<clipPath id="terminal-3189088113-line-17">
     <rect x="0" y="416.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-18">
+<clipPath id="terminal-3189088113-line-18">
     <rect x="0" y="440.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-19">
+<clipPath id="terminal-3189088113-line-19">
     <rect x="0" y="465.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-20">
+<clipPath id="terminal-3189088113-line-20">
     <rect x="0" y="489.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-21">
+<clipPath id="terminal-3189088113-line-21">
     <rect x="0" y="513.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-22">
+<clipPath id="terminal-3189088113-line-22">
     <rect x="0" y="538.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-23">
+<clipPath id="terminal-3189088113-line-23">
     <rect x="0" y="562.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-24">
+<clipPath id="terminal-3189088113-line-24">
     <rect x="0" y="587.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-25">
+<clipPath id="terminal-3189088113-line-25">
     <rect x="0" y="611.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-26">
+<clipPath id="terminal-3189088113-line-26">
     <rect x="0" y="635.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-27">
+<clipPath id="terminal-3189088113-line-27">
     <rect x="0" y="660.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-28">
+<clipPath id="terminal-3189088113-line-28">
     <rect x="0" y="684.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-29">
+<clipPath id="terminal-3189088113-line-29">
     <rect x="0" y="709.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-30">
+<clipPath id="terminal-3189088113-line-30">
     <rect x="0" y="733.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-31">
+<clipPath id="terminal-3189088113-line-31">
     <rect x="0" y="757.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-32">
+<clipPath id="terminal-3189088113-line-32">
     <rect x="0" y="782.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-33">
+<clipPath id="terminal-3189088113-line-33">
     <rect x="0" y="806.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-34">
+<clipPath id="terminal-3189088113-line-34">
     <rect x="0" y="831.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-35">
+<clipPath id="terminal-3189088113-line-35">
     <rect x="0" y="855.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-36">
+<clipPath id="terminal-3189088113-line-36">
     <rect x="0" y="879.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-37">
+<clipPath id="terminal-3189088113-line-37">
     <rect x="0" y="904.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-38">
+<clipPath id="terminal-3189088113-line-38">
     <rect x="0" y="928.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-39">
+<clipPath id="terminal-3189088113-line-39">
     <rect x="0" y="953.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-40">
+<clipPath id="terminal-3189088113-line-40">
     <rect x="0" y="977.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-41">
+<clipPath id="terminal-3189088113-line-41">
     <rect x="0" y="1001.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-42">
+<clipPath id="terminal-3189088113-line-42">
     <rect x="0" y="1026.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-43">
+<clipPath id="terminal-3189088113-line-43">
     <rect x="0" y="1050.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-44">
+<clipPath id="terminal-3189088113-line-44">
     <rect x="0" y="1075.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-45">
+<clipPath id="terminal-3189088113-line-45">
     <rect x="0" y="1099.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-46">
+<clipPath id="terminal-3189088113-line-46">
     <rect x="0" y="1123.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-47">
+<clipPath id="terminal-3189088113-line-47">
     <rect x="0" y="1148.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-48">
+<clipPath id="terminal-3189088113-line-48">
     <rect x="0" y="1172.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-49">
+<clipPath id="terminal-3189088113-line-49">
     <rect x="0" y="1197.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-50">
+<clipPath id="terminal-3189088113-line-50">
     <rect x="0" y="1221.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-51">
+<clipPath id="terminal-3189088113-line-51">
     <rect x="0" y="1245.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-52">
+<clipPath id="terminal-3189088113-line-52">
     <rect x="0" y="1270.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-1803387802-line-53">
+<clipPath id="terminal-3189088113-line-53">
     <rect x="0" y="1294.7" width="1464" height="24.65"/>
             </clipPath>
+<clipPath id="terminal-3189088113-line-54">
+    <rect x="0" y="1319.1" width="1464" height="24.65"/>
+            </clipPath>
     </defs>
 
-    <rect fill="#292929" stroke="rgba(255,255,255,0.35)" stroke-width="1" x="1" y="1" width="1480" height="1390" rx="8"/><text class="terminal-1803387802-title" fill="#c5c8c6" text-anchor="middle" x="740" y="27">Command:&#160;build-docs</text>
+    <rect fill="#292929" stroke="rgba(255,255,255,0.35)" stroke-width="1" x="1" y="1" width="1480" height="1414.4" rx="8"/><text class="terminal-3189088113-title" fill="#c5c8c6" text-anchor="middle" x="740" y="27">Command:&#160;build-docs</text>
             <g transform="translate(26,22)">
             <circle cx="0" cy="0" r="7" fill="#ff5f57"/>
             <circle cx="22" cy="0" r="7" fill="#febc2e"/>
             <circle cx="44" cy="0" r="7" fill="#28c840"/>
             </g>
         
-    <g transform="translate(9, 41)" clip-path="url(#terminal-1803387802-clip-terminal)">
+    <g transform="translate(9, 41)" clip-path="url(#terminal-3189088113-clip-terminal)">
     
-    <g class="terminal-1803387802-matrix">
-    <text class="terminal-1803387802-r2" x="1464" y="20" textLength="12.2" clip-path="url(#terminal-1803387802-line-0)">
-</text><text class="terminal-1803387802-r3" x="12.2" y="44.4" textLength="85.4" clip-path="url(#terminal-1803387802-line-1)">Usage:&#160;</text><text class="terminal-1803387802-r1" x="97.6" y="44.4" textLength="329.4" clip-path="url(#terminal-1803387802-line-1)">breeze&#160;build-docs&#160;[OPTIONS]</text><text class="terminal-1803387802-r2" x="1464" y="44.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-1)">
-</text><text class="terminal-1803387802-r2" x="1464" y="68.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-2)">
-</text><text class="terminal-1803387802-r2" x="12.2" y="93.2" textLength="451.4" clip-path="url(#terminal-1803387802-line-3)">Build&#160;documentation&#160;in&#160;the&#160;container.</text><text class="terminal-1803387802-r2" x="1464" y="93.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-3)">
-</text><text class="terminal-1803387802-r2" x="1464" y="117.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-4)">
-</text><text class="terminal-1803387802-r4" x="0" y="142" textLength="24.4" clip-path="url(#terminal-1803387802-line-5)">╭─</text><text class="terminal-1803387802-r4" x="24.4" y="142" textLength="1415.2" clip-path="url(#terminal-1803387802-line-5)">&#160;Doc&#160;flags&#160;─────────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-1803387802-r4" x="1439.6" y="142" textLength="24.4" clip-path="url(#terminal-180338 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="166.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-6)">│</text><text class="terminal-1803387802-r5" x="24.4" y="166.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-6)">-</text><text class="terminal-1803387802-r5" x="36.6" y="166.4" textLength="61" clip-path="url(#terminal-1803387802-line-6)">-docs</text><text class="terminal-1803387802-r5" x="97.6" y="166.4" textLength="61" clip-path="url(#terminal-18033878 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="190.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-7)">│</text><text class="terminal-1803387802-r5" x="24.4" y="190.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-7)">-</text><text class="terminal-1803387802-r5" x="36.6" y="190.8" textLength="134.2" clip-path="url(#terminal-1803387802-line-7)">-spellcheck</text><text class="terminal-1803387802-r5" x="170.8" y="190.8" textLength="61" clip-path="url(#termina [...]
-</text><text class="terminal-1803387802-r4" x="0" y="215.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-8)">│</text><text class="terminal-1803387802-r5" x="24.4" y="215.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-8)">-</text><text class="terminal-1803387802-r5" x="36.6" y="215.2" textLength="48.8" clip-path="url(#terminal-1803387802-line-8)">-for</text><text class="terminal-1803387802-r5" x="85.4" y="215.2" textLength="134.2" clip-path="url(#terminal-1803 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="239.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-9)">│</text><text class="terminal-1803387802-r5" x="24.4" y="239.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-9)">-</text><text class="terminal-1803387802-r5" x="36.6" y="239.6" textLength="97.6" clip-path="url(#terminal-1803387802-line-9)">-package</text><text class="terminal-1803387802-r5" x="134.2" y="239.6" textLength="85.4" clip-path="url(#terminal- [...]
-</text><text class="terminal-1803387802-r4" x="0" y="264" textLength="12.2" clip-path="url(#terminal-1803387802-line-10)">│</text><text class="terminal-1803387802-r7" x="305" y="264" textLength="1134.6" clip-path="url(#terminal-1803387802-line-10)">(apache-airflow&#160;|&#160;apache-airflow-providers&#160;|&#160;apache-airflow-providers-airbyte&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451.8"  [...]
-</text><text class="terminal-1803387802-r4" x="0" y="288.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-11)">│</text><text class="terminal-1803387802-r7" x="305" y="288.4" textLength="1134.6" clip-path="url(#terminal-1803387802-line-11)">apache-airflow-providers-alibaba&#160;|&#160;apache-airflow-providers-amazon&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text [...]
-</text><text class="terminal-1803387802-r4" x="0" y="312.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-12)">│</text><text class="terminal-1803387802-r7" x="305" y="312.8" textLength="1134.6" clip-path="url(#terminal-1803387802-line-12)">apache-airflow-providers-apache-beam&#160;|&#160;apache-airflow-providers-apache-cassandra&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451.8" y="312.8" textLength= [...]
-</text><text class="terminal-1803387802-r4" x="0" y="337.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-13)">│</text><text class="terminal-1803387802-r7" x="305" y="337.2" textLength="1134.6" clip-path="url(#terminal-1803387802-line-13)">apache-airflow-providers-apache-drill&#160;|&#160;apache-airflow-providers-apache-druid&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451.8" y="337 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="361.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-14)">│</text><text class="terminal-1803387802-r7" x="305" y="361.6" textLength="1134.6" clip-path="url(#terminal-1803387802-line-14)">apache-airflow-providers-apache-hdfs&#160;|&#160;apache-airflow-providers-apache-hive&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="386" textLength="12.2" clip-path="url(#terminal-1803387802-line-15)">│</text><text class="terminal-1803387802-r7" x="305" y="386" textLength="1134.6" clip-path="url(#terminal-1803387802-line-15)">apache-airflow-providers-apache-kylin&#160;|&#160;apache-airflow-providers-apache-livy&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451.8" y="38 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="410.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-16)">│</text><text class="terminal-1803387802-r7" x="305" y="410.4" textLength="1134.6" clip-path="url(#terminal-1803387802-line-16)">apache-airflow-providers-apache-pig&#160;|&#160;apache-airflow-providers-apache-pinot&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="434.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-17)">│</text><text class="terminal-1803387802-r7" x="305" y="434.8" textLength="1134.6" clip-path="url(#terminal-1803387802-line-17)">apache-airflow-providers-apache-spark&#160;|&#160;apache-airflow-providers-apache-sqoop&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451.8" y="434 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="459.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-18)">│</text><text class="terminal-1803387802-r7" x="305" y="459.2" textLength="1134.6" clip-path="url(#terminal-1803387802-line-18)">apache-airflow-providers-arangodb&#160;|&#160;apache-airflow-providers-asana&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text [...]
-</text><text class="terminal-1803387802-r4" x="0" y="483.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-19)">│</text><text class="terminal-1803387802-r7" x="305" y="483.6" textLength="1134.6" clip-path="url(#terminal-1803387802-line-19)">apache-airflow-providers-celery&#160;|&#160;apache-airflow-providers-cloudant&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><tex [...]
-</text><text class="terminal-1803387802-r4" x="0" y="508" textLength="12.2" clip-path="url(#terminal-1803387802-line-20)">│</text><text class="terminal-1803387802-r7" x="305" y="508" textLength="1134.6" clip-path="url(#terminal-1803387802-line-20)">apache-airflow-providers-cncf-kubernetes&#160;|&#160;apache-airflow-providers-databricks&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451.8" y="508" textLen [...]
-</text><text class="terminal-1803387802-r4" x="0" y="532.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-21)">│</text><text class="terminal-1803387802-r7" x="305" y="532.4" textLength="1134.6" clip-path="url(#terminal-1803387802-line-21)">apache-airflow-providers-datadog&#160;|&#160;apache-airflow-providers-dbt-cloud&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="t [...]
-</text><text class="terminal-1803387802-r4" x="0" y="556.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-22)">│</text><text class="terminal-1803387802-r7" x="305" y="556.8" textLength="1134.6" clip-path="url(#terminal-1803387802-line-22)">apache-airflow-providers-dingding&#160;|&#160;apache-airflow-providers-discord&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text cla [...]
-</text><text class="terminal-1803387802-r4" x="0" y="581.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-23)">│</text><text class="terminal-1803387802-r7" x="305" y="581.2" textLength="1134.6" clip-path="url(#terminal-1803387802-line-23)">apache-airflow-providers-docker&#160;|&#160;apache-airflow-providers-elasticsearch&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="605.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-24)">│</text><text class="terminal-1803387802-r7" x="305" y="605.6" textLength="1134.6" clip-path="url(#terminal-1803387802-line-24)">apache-airflow-providers-exasol&#160;|&#160;apache-airflow-providers-facebook&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><tex [...]
-</text><text class="terminal-1803387802-r4" x="0" y="630" textLength="12.2" clip-path="url(#terminal-1803387802-line-25)">│</text><text class="terminal-1803387802-r7" x="305" y="630" textLength="1134.6" clip-path="url(#terminal-1803387802-line-25)">apache-airflow-providers-ftp&#160;|&#160;apache-airflow-providers-github&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&# [...]
-</text><text class="terminal-1803387802-r4" x="0" y="654.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-26)">│</text><text class="terminal-1803387802-r7" x="305" y="654.4" textLength="1134.6" clip-path="url(#terminal-1803387802-line-26)">apache-airflow-providers-google&#160;|&#160;apache-airflow-providers-grpc&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#1 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="678.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-27)">│</text><text class="terminal-1803387802-r7" x="305" y="678.8" textLength="1134.6" clip-path="url(#terminal-1803387802-line-27)">apache-airflow-providers-hashicorp&#160;|&#160;apache-airflow-providers-http&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text [...]
-</text><text class="terminal-1803387802-r4" x="0" y="703.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-28)">│</text><text class="terminal-1803387802-r7" x="305" y="703.2" textLength="1134.6" clip-path="url(#terminal-1803387802-line-28)">apache-airflow-providers-imap&#160;|&#160;apache-airflow-providers-influxdb&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;< [...]
-</text><text class="terminal-1803387802-r4" x="0" y="727.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-29)">│</text><text class="terminal-1803387802-r7" x="305" y="727.6" textLength="1134.6" clip-path="url(#terminal-1803387802-line-29)">apache-airflow-providers-jdbc&#160;|&#160;apache-airflow-providers-jenkins&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&# [...]
-</text><text class="terminal-1803387802-r4" x="0" y="752" textLength="12.2" clip-path="url(#terminal-1803387802-line-30)">│</text><text class="terminal-1803387802-r7" x="305" y="752" textLength="1134.6" clip-path="url(#terminal-1803387802-line-30)">apache-airflow-providers-jira&#160;|&#160;apache-airflow-providers-microsoft-azure&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802- [...]
-</text><text class="terminal-1803387802-r4" x="0" y="776.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-31)">│</text><text class="terminal-1803387802-r7" x="305" y="776.4" textLength="1134.6" clip-path="url(#terminal-1803387802-line-31)">apache-airflow-providers-microsoft-mssql&#160;|&#160;apache-airflow-providers-microsoft-psrp&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r4" x="1451.8" y="776.4" textLength="12.2" cli [...]
-</text><text class="terminal-1803387802-r4" x="0" y="800.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-32)">│</text><text class="terminal-1803387802-r7" x="305" y="800.8" textLength="1134.6" clip-path="url(#terminal-1803387802-line-32)">apache-airflow-providers-microsoft-winrm&#160;|&#160;apache-airflow-providers-mongo&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-1803387802-r [...]
-</text><text class="terminal-1803387802-r4" x="0" y="825.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-33)">│</text><text class="terminal-1803387802-r7" x="305" y="825.2" textLength="1134.6" clip-path="url(#terminal-1803387802-line-33)">apache-airflow-providers-mysql&#160;|&#160;apache-airflow-providers-neo4j&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#1 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="849.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-34)">│</text><text class="terminal-1803387802-r7" x="305" y="849.6" textLength="1134.6" clip-path="url(#terminal-1803387802-line-34)">apache-airflow-providers-odbc&#160;|&#160;apache-airflow-providers-openfaas&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;< [...]
-</text><text class="terminal-1803387802-r4" x="0" y="874" textLength="12.2" clip-path="url(#terminal-1803387802-line-35)">│</text><text class="terminal-1803387802-r7" x="305" y="874" textLength="1134.6" clip-path="url(#terminal-1803387802-line-35)">apache-airflow-providers-opsgenie&#160;|&#160;apache-airflow-providers-oracle&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text cl [...]
-</text><text class="terminal-1803387802-r4" x="0" y="898.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-36)">│</text><text class="terminal-1803387802-r7" x="305" y="898.4" textLength="1134.6" clip-path="url(#terminal-1803387802-line-36)">apache-airflow-providers-pagerduty&#160;|&#160;apache-airflow-providers-papermill&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-18 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="922.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-37)">│</text><text class="terminal-1803387802-r7" x="305" y="922.8" textLength="1134.6" clip-path="url(#terminal-1803387802-line-37)">apache-airflow-providers-plexus&#160;|&#160;apache-airflow-providers-postgres&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><tex [...]
-</text><text class="terminal-1803387802-r4" x="0" y="947.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-38)">│</text><text class="terminal-1803387802-r7" x="305" y="947.2" textLength="1134.6" clip-path="url(#terminal-1803387802-line-38)">apache-airflow-providers-presto&#160;|&#160;apache-airflow-providers-qubole&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;< [...]
-</text><text class="terminal-1803387802-r4" x="0" y="971.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-39)">│</text><text class="terminal-1803387802-r7" x="305" y="971.6" textLength="1134.6" clip-path="url(#terminal-1803387802-line-39)">apache-airflow-providers-redis&#160;|&#160;apache-airflow-providers-salesforce&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text cla [...]
-</text><text class="terminal-1803387802-r4" x="0" y="996" textLength="12.2" clip-path="url(#terminal-1803387802-line-40)">│</text><text class="terminal-1803387802-r7" x="305" y="996" textLength="1134.6" clip-path="url(#terminal-1803387802-line-40)">apache-airflow-providers-samba&#160;|&#160;apache-airflow-providers-segment&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</tex [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1020.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-41)">│</text><text class="terminal-1803387802-r7" x="305" y="1020.4" textLength="1134.6" clip-path="url(#terminal-1803387802-line-41)">apache-airflow-providers-sendgrid&#160;|&#160;apache-airflow-providers-sftp&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1044.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-42)">│</text><text class="terminal-1803387802-r7" x="305" y="1044.8" textLength="1134.6" clip-path="url(#terminal-1803387802-line-42)">apache-airflow-providers-singularity&#160;|&#160;apache-airflow-providers-slack&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class= [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1069.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-43)">│</text><text class="terminal-1803387802-r7" x="305" y="1069.2" textLength="1134.6" clip-path="url(#terminal-1803387802-line-43)">apache-airflow-providers-snowflake&#160;|&#160;apache-airflow-providers-sqlite&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text c [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1093.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-44)">│</text><text class="terminal-1803387802-r7" x="305" y="1093.6" textLength="1134.6" clip-path="url(#terminal-1803387802-line-44)">apache-airflow-providers-ssh&#160;|&#160;apache-airflow-providers-tableau&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;& [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1118" textLength="12.2" clip-path="url(#terminal-1803387802-line-45)">│</text><text class="terminal-1803387802-r7" x="305" y="1118" textLength="1134.6" clip-path="url(#terminal-1803387802-line-45)">apache-airflow-providers-telegram&#160;|&#160;apache-airflow-providers-trino&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text>< [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1142.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-46)">│</text><text class="terminal-1803387802-r7" x="305" y="1142.4" textLength="1134.6" clip-path="url(#terminal-1803387802-line-46)">apache-airflow-providers-vertica&#160;|&#160;apache-airflow-providers-yandex&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</te [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1166.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-47)">│</text><text class="terminal-1803387802-r7" x="305" y="1166.8" textLength="1134.6" clip-path="url(#terminal-1803387802-line-47)">apache-airflow-providers-zendesk&#160;|&#160;docker-stack&#160;|&#160;helm-chart)&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160; [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1191.2" textLength="1464" clip-path="url(#terminal-1803387802-line-48)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-1803387802-r2" x="1464" y="1191.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-48)">
-</text><text class="terminal-1803387802-r4" x="0" y="1215.6" textLength="24.4" clip-path="url(#terminal-1803387802-line-49)">╭─</text><text class="terminal-1803387802-r4" x="24.4" y="1215.6" textLength="1415.2" clip-path="url(#terminal-1803387802-line-49)">&#160;Options&#160;───────────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-1803387802-r4" x="1439.6" y="1215.6" textLength="24.4" clip-path="url(#terminal- [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1240" textLength="12.2" clip-path="url(#terminal-1803387802-line-50)">│</text><text class="terminal-1803387802-r5" x="24.4" y="1240" textLength="12.2" clip-path="url(#terminal-1803387802-line-50)">-</text><text class="terminal-1803387802-r5" x="36.6" y="1240" textLength="97.6" clip-path="url(#terminal-1803387802-line-50)">-verbose</text><text class="terminal-1803387802-r6" x="280.6" y="1240" textLength="24.4" clip-path="url(#terminal-1 [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1264.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-51)">│</text><text class="terminal-1803387802-r5" x="24.4" y="1264.4" textLength="12.2" clip-path="url(#terminal-1803387802-line-51)">-</text><text class="terminal-1803387802-r5" x="36.6" y="1264.4" textLength="48.8" clip-path="url(#terminal-1803387802-line-51)">-dry</text><text class="terminal-1803387802-r5" x="85.4" y="1264.4" textLength="48.8" clip-path="url(#termina [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1288.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-52)">│</text><text class="terminal-1803387802-r5" x="24.4" y="1288.8" textLength="12.2" clip-path="url(#terminal-1803387802-line-52)">-</text><text class="terminal-1803387802-r5" x="36.6" y="1288.8" textLength="85.4" clip-path="url(#terminal-1803387802-line-52)">-github</text><text class="terminal-1803387802-r5" x="122" y="1288.8" textLength="134.2" clip-path="url(#term [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1313.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-53)">│</text><text class="terminal-1803387802-r5" x="24.4" y="1313.2" textLength="12.2" clip-path="url(#terminal-1803387802-line-53)">-</text><text class="terminal-1803387802-r5" x="36.6" y="1313.2" textLength="61" clip-path="url(#terminal-1803387802-line-53)">-help</text><text class="terminal-1803387802-r6" x="280.6" y="1313.2" textLength="24.4" clip-path="url(#termina [...]
-</text><text class="terminal-1803387802-r4" x="0" y="1337.6" textLength="1464" clip-path="url(#terminal-1803387802-line-54)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-1803387802-r2" x="1464" y="1337.6" textLength="12.2" clip-path="url(#terminal-1803387802-line-54)">
+    <g class="terminal-3189088113-matrix">
+    <text class="terminal-3189088113-r2" x="1464" y="20" textLength="12.2" clip-path="url(#terminal-3189088113-line-0)">
+</text><text class="terminal-3189088113-r3" x="12.2" y="44.4" textLength="85.4" clip-path="url(#terminal-3189088113-line-1)">Usage:&#160;</text><text class="terminal-3189088113-r1" x="97.6" y="44.4" textLength="329.4" clip-path="url(#terminal-3189088113-line-1)">breeze&#160;build-docs&#160;[OPTIONS]</text><text class="terminal-3189088113-r2" x="1464" y="44.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-1)">
+</text><text class="terminal-3189088113-r2" x="1464" y="68.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-2)">
+</text><text class="terminal-3189088113-r2" x="12.2" y="93.2" textLength="451.4" clip-path="url(#terminal-3189088113-line-3)">Build&#160;documentation&#160;in&#160;the&#160;container.</text><text class="terminal-3189088113-r2" x="1464" y="93.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-3)">
+</text><text class="terminal-3189088113-r2" x="1464" y="117.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-4)">
+</text><text class="terminal-3189088113-r4" x="0" y="142" textLength="24.4" clip-path="url(#terminal-3189088113-line-5)">╭─</text><text class="terminal-3189088113-r4" x="24.4" y="142" textLength="1415.2" clip-path="url(#terminal-3189088113-line-5)">&#160;Doc&#160;flags&#160;─────────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-3189088113-r4" x="1439.6" y="142" textLength="24.4" clip-path="url(#terminal-318908 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="166.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-6)">│</text><text class="terminal-3189088113-r5" x="24.4" y="166.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-6)">-</text><text class="terminal-3189088113-r5" x="36.6" y="166.4" textLength="61" clip-path="url(#terminal-3189088113-line-6)">-docs</text><text class="terminal-3189088113-r5" x="97.6" y="166.4" textLength="61" clip-path="url(#terminal-31890881 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="190.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-7)">│</text><text class="terminal-3189088113-r5" x="24.4" y="190.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-7)">-</text><text class="terminal-3189088113-r5" x="36.6" y="190.8" textLength="134.2" clip-path="url(#terminal-3189088113-line-7)">-spellcheck</text><text class="terminal-3189088113-r5" x="170.8" y="190.8" textLength="61" clip-path="url(#termina [...]
+</text><text class="terminal-3189088113-r4" x="0" y="215.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-8)">│</text><text class="terminal-3189088113-r5" x="24.4" y="215.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-8)">-</text><text class="terminal-3189088113-r5" x="36.6" y="215.2" textLength="48.8" clip-path="url(#terminal-3189088113-line-8)">-for</text><text class="terminal-3189088113-r5" x="85.4" y="215.2" textLength="134.2" clip-path="url(#terminal-3189 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="239.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-9)">│</text><text class="terminal-3189088113-r5" x="24.4" y="239.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-9)">-</text><text class="terminal-3189088113-r5" x="36.6" y="239.6" textLength="97.6" clip-path="url(#terminal-3189088113-line-9)">-package</text><text class="terminal-3189088113-r5" x="134.2" y="239.6" textLength="85.4" clip-path="url(#terminal- [...]
+</text><text class="terminal-3189088113-r4" x="0" y="264" textLength="12.2" clip-path="url(#terminal-3189088113-line-10)">│</text><text class="terminal-3189088113-r7" x="305" y="264" textLength="1134.6" clip-path="url(#terminal-3189088113-line-10)">(apache-airflow&#160;|&#160;apache-airflow-providers&#160;|&#160;apache-airflow-providers-airbyte&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8"  [...]
+</text><text class="terminal-3189088113-r4" x="0" y="288.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-11)">│</text><text class="terminal-3189088113-r7" x="305" y="288.4" textLength="1134.6" clip-path="url(#terminal-3189088113-line-11)">apache-airflow-providers-alibaba&#160;|&#160;apache-airflow-providers-amazon&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text [...]
+</text><text class="terminal-3189088113-r4" x="0" y="312.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-12)">│</text><text class="terminal-3189088113-r7" x="305" y="312.8" textLength="1134.6" clip-path="url(#terminal-3189088113-line-12)">apache-airflow-providers-apache-beam&#160;|&#160;apache-airflow-providers-apache-cassandra&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8" y="312.8" textLength= [...]
+</text><text class="terminal-3189088113-r4" x="0" y="337.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-13)">│</text><text class="terminal-3189088113-r7" x="305" y="337.2" textLength="1134.6" clip-path="url(#terminal-3189088113-line-13)">apache-airflow-providers-apache-drill&#160;|&#160;apache-airflow-providers-apache-druid&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8" y="337 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="361.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-14)">│</text><text class="terminal-3189088113-r7" x="305" y="361.6" textLength="1134.6" clip-path="url(#terminal-3189088113-line-14)">apache-airflow-providers-apache-hdfs&#160;|&#160;apache-airflow-providers-apache-hive&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="386" textLength="12.2" clip-path="url(#terminal-3189088113-line-15)">│</text><text class="terminal-3189088113-r7" x="305" y="386" textLength="1134.6" clip-path="url(#terminal-3189088113-line-15)">apache-airflow-providers-apache-kylin&#160;|&#160;apache-airflow-providers-apache-livy&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8" y="38 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="410.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-16)">│</text><text class="terminal-3189088113-r7" x="305" y="410.4" textLength="1134.6" clip-path="url(#terminal-3189088113-line-16)">apache-airflow-providers-apache-pig&#160;|&#160;apache-airflow-providers-apache-pinot&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="434.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-17)">│</text><text class="terminal-3189088113-r7" x="305" y="434.8" textLength="1134.6" clip-path="url(#terminal-3189088113-line-17)">apache-airflow-providers-apache-spark&#160;|&#160;apache-airflow-providers-apache-sqoop&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8" y="434 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="459.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-18)">│</text><text class="terminal-3189088113-r7" x="305" y="459.2" textLength="1134.6" clip-path="url(#terminal-3189088113-line-18)">apache-airflow-providers-arangodb&#160;|&#160;apache-airflow-providers-asana&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text [...]
+</text><text class="terminal-3189088113-r4" x="0" y="483.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-19)">│</text><text class="terminal-3189088113-r7" x="305" y="483.6" textLength="1134.6" clip-path="url(#terminal-3189088113-line-19)">apache-airflow-providers-celery&#160;|&#160;apache-airflow-providers-cloudant&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><tex [...]
+</text><text class="terminal-3189088113-r4" x="0" y="508" textLength="12.2" clip-path="url(#terminal-3189088113-line-20)">│</text><text class="terminal-3189088113-r7" x="305" y="508" textLength="1134.6" clip-path="url(#terminal-3189088113-line-20)">apache-airflow-providers-cncf-kubernetes&#160;|&#160;apache-airflow-providers-core-sql&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8" y="50 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="532.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-21)">│</text><text class="terminal-3189088113-r7" x="305" y="532.4" textLength="1134.6" clip-path="url(#terminal-3189088113-line-21)">apache-airflow-providers-databricks&#160;|&#160;apache-airflow-providers-datadog&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="termin [...]
+</text><text class="terminal-3189088113-r4" x="0" y="556.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-22)">│</text><text class="terminal-3189088113-r7" x="305" y="556.8" textLength="1134.6" clip-path="url(#terminal-3189088113-line-22)">apache-airflow-providers-dbt-cloud&#160;|&#160;apache-airflow-providers-dingding&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="termin [...]
+</text><text class="terminal-3189088113-r4" x="0" y="581.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-23)">│</text><text class="terminal-3189088113-r7" x="305" y="581.2" textLength="1134.6" clip-path="url(#terminal-3189088113-line-23)">apache-airflow-providers-discord&#160;|&#160;apache-airflow-providers-docker&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text [...]
+</text><text class="terminal-3189088113-r4" x="0" y="605.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-24)">│</text><text class="terminal-3189088113-r7" x="305" y="605.6" textLength="1134.6" clip-path="url(#terminal-3189088113-line-24)">apache-airflow-providers-elasticsearch&#160;|&#160;apache-airflow-providers-exasol&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="630" textLength="12.2" clip-path="url(#terminal-3189088113-line-25)">│</text><text class="terminal-3189088113-r7" x="305" y="630" textLength="1134.6" clip-path="url(#terminal-3189088113-line-25)">apache-airflow-providers-facebook&#160;|&#160;apache-airflow-providers-ftp&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160; [...]
+</text><text class="terminal-3189088113-r4" x="0" y="654.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-26)">│</text><text class="terminal-3189088113-r7" x="305" y="654.4" textLength="1134.6" clip-path="url(#terminal-3189088113-line-26)">apache-airflow-providers-github&#160;|&#160;apache-airflow-providers-google&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;< [...]
+</text><text class="terminal-3189088113-r4" x="0" y="678.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-27)">│</text><text class="terminal-3189088113-r7" x="305" y="678.8" textLength="1134.6" clip-path="url(#terminal-3189088113-line-27)">apache-airflow-providers-grpc&#160;|&#160;apache-airflow-providers-hashicorp&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text [...]
+</text><text class="terminal-3189088113-r4" x="0" y="703.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-28)">│</text><text class="terminal-3189088113-r7" x="305" y="703.2" textLength="1134.6" clip-path="url(#terminal-3189088113-line-28)">apache-airflow-providers-http&#160;|&#160;apache-airflow-providers-imap&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="727.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-29)">│</text><text class="terminal-3189088113-r7" x="305" y="727.6" textLength="1134.6" clip-path="url(#terminal-3189088113-line-29)">apache-airflow-providers-influxdb&#160;|&#160;apache-airflow-providers-jdbc&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;< [...]
+</text><text class="terminal-3189088113-r4" x="0" y="752" textLength="12.2" clip-path="url(#terminal-3189088113-line-30)">│</text><text class="terminal-3189088113-r7" x="305" y="752" textLength="1134.6" clip-path="url(#terminal-3189088113-line-30)">apache-airflow-providers-jenkins&#160;|&#160;apache-airflow-providers-jira&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160; [...]
+</text><text class="terminal-3189088113-r4" x="0" y="776.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-31)">│</text><text class="terminal-3189088113-r7" x="305" y="776.4" textLength="1134.6" clip-path="url(#terminal-3189088113-line-31)">apache-airflow-providers-microsoft-azure&#160;|&#160;apache-airflow-providers-microsoft-mssql&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8" y="776.4" textLength="12.2" clip-pat [...]
+</text><text class="terminal-3189088113-r4" x="0" y="800.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-32)">│</text><text class="terminal-3189088113-r7" x="305" y="800.8" textLength="1134.6" clip-path="url(#terminal-3189088113-line-32)">apache-airflow-providers-microsoft-psrp&#160;|&#160;apache-airflow-providers-microsoft-winrm&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8" y="800.8" textLength="12.2" cli [...]
+</text><text class="terminal-3189088113-r4" x="0" y="825.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-33)">│</text><text class="terminal-3189088113-r7" x="305" y="825.2" textLength="1134.6" clip-path="url(#terminal-3189088113-line-33)">apache-airflow-providers-mongo&#160;|&#160;apache-airflow-providers-mysql&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#1 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="849.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-34)">│</text><text class="terminal-3189088113-r7" x="305" y="849.6" textLength="1134.6" clip-path="url(#terminal-3189088113-line-34)">apache-airflow-providers-neo4j&#160;|&#160;apache-airflow-providers-odbc&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#16 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="874" textLength="12.2" clip-path="url(#terminal-3189088113-line-35)">│</text><text class="terminal-3189088113-r7" x="305" y="874" textLength="1134.6" clip-path="url(#terminal-3189088113-line-35)">apache-airflow-providers-openfaas&#160;|&#160;apache-airflow-providers-opsgenie&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="termi [...]
+</text><text class="terminal-3189088113-r4" x="0" y="898.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-36)">│</text><text class="terminal-3189088113-r7" x="305" y="898.4" textLength="1134.6" clip-path="url(#terminal-3189088113-line-36)">apache-airflow-providers-oracle&#160;|&#160;apache-airflow-providers-pagerduty&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text cla [...]
+</text><text class="terminal-3189088113-r4" x="0" y="922.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-37)">│</text><text class="terminal-3189088113-r7" x="305" y="922.8" textLength="1134.6" clip-path="url(#terminal-3189088113-line-37)">apache-airflow-providers-papermill&#160;|&#160;apache-airflow-providers-plexus&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text cla [...]
+</text><text class="terminal-3189088113-r4" x="0" y="947.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-38)">│</text><text class="terminal-3189088113-r7" x="305" y="947.2" textLength="1134.6" clip-path="url(#terminal-3189088113-line-38)">apache-airflow-providers-postgres&#160;|&#160;apache-airflow-providers-presto&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><tex [...]
+</text><text class="terminal-3189088113-r4" x="0" y="971.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-39)">│</text><text class="terminal-3189088113-r7" x="305" y="971.6" textLength="1134.6" clip-path="url(#terminal-3189088113-line-39)">apache-airflow-providers-qubole&#160;|&#160;apache-airflow-providers-redis&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&# [...]
+</text><text class="terminal-3189088113-r4" x="0" y="996" textLength="12.2" clip-path="url(#terminal-3189088113-line-40)">│</text><text class="terminal-3189088113-r7" x="305" y="996" textLength="1134.6" clip-path="url(#terminal-3189088113-line-40)">apache-airflow-providers-salesforce&#160;|&#160;apache-airflow-providers-samba&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class=" [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1020.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-41)">│</text><text class="terminal-3189088113-r7" x="305" y="1020.4" textLength="1134.6" clip-path="url(#terminal-3189088113-line-41)">apache-airflow-providers-segment&#160;|&#160;apache-airflow-providers-sendgrid&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text c [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1044.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-42)">│</text><text class="terminal-3189088113-r7" x="305" y="1044.8" textLength="1134.6" clip-path="url(#terminal-3189088113-line-42)">apache-airflow-providers-sftp&#160;|&#160;apache-airflow-providers-singularity&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text c [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1069.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-43)">│</text><text class="terminal-3189088113-r7" x="305" y="1069.2" textLength="1134.6" clip-path="url(#terminal-3189088113-line-43)">apache-airflow-providers-slack&#160;|&#160;apache-airflow-providers-snowflake&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><t [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1093.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-44)">│</text><text class="terminal-3189088113-r7" x="305" y="1093.6" textLength="1134.6" clip-path="url(#terminal-3189088113-line-44)">apache-airflow-providers-sqlite&#160;|&#160;apache-airflow-providers-ssh&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&# [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1118" textLength="12.2" clip-path="url(#terminal-3189088113-line-45)">│</text><text class="terminal-3189088113-r7" x="305" y="1118" textLength="1134.6" clip-path="url(#terminal-3189088113-line-45)">apache-airflow-providers-tableau&#160;|&#160;apache-airflow-providers-telegram&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1142.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-46)">│</text><text class="terminal-3189088113-r7" x="305" y="1142.4" textLength="1134.6" clip-path="url(#terminal-3189088113-line-46)">apache-airflow-providers-trino&#160;|&#160;apache-airflow-providers-vertica&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1166.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-47)">│</text><text class="terminal-3189088113-r7" x="305" y="1166.8" textLength="1134.6" clip-path="url(#terminal-3189088113-line-47)">apache-airflow-providers-yandex&#160;|&#160;apache-airflow-providers-zendesk&#160;|&#160;docker-stack&#160;|&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;</text><text class="terminal-3189088113-r4" x="1451.8" y="1166.8" tex [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1191.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-48)">│</text><text class="terminal-3189088113-r7" x="305" y="1191.2" textLength="1134.6" clip-path="url(#terminal-3189088113-line-48)">helm-chart)&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#16 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1215.6" textLength="1464" clip-path="url(#terminal-3189088113-line-49)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-3189088113-r2" x="1464" y="1215.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-49)">
+</text><text class="terminal-3189088113-r4" x="0" y="1240" textLength="24.4" clip-path="url(#terminal-3189088113-line-50)">╭─</text><text class="terminal-3189088113-r4" x="24.4" y="1240" textLength="1415.2" clip-path="url(#terminal-3189088113-line-50)">&#160;Options&#160;───────────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-3189088113-r4" x="1439.6" y="1240" textLength="24.4" clip-path="url(#terminal-318908 [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1264.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-51)">│</text><text class="terminal-3189088113-r5" x="24.4" y="1264.4" textLength="12.2" clip-path="url(#terminal-3189088113-line-51)">-</text><text class="terminal-3189088113-r5" x="36.6" y="1264.4" textLength="97.6" clip-path="url(#terminal-3189088113-line-51)">-verbose</text><text class="terminal-3189088113-r6" x="280.6" y="1264.4" textLength="24.4" clip-path="url(#te [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1288.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-52)">│</text><text class="terminal-3189088113-r5" x="24.4" y="1288.8" textLength="12.2" clip-path="url(#terminal-3189088113-line-52)">-</text><text class="terminal-3189088113-r5" x="36.6" y="1288.8" textLength="48.8" clip-path="url(#terminal-3189088113-line-52)">-dry</text><text class="terminal-3189088113-r5" x="85.4" y="1288.8" textLength="48.8" clip-path="url(#termina [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1313.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-53)">│</text><text class="terminal-3189088113-r5" x="24.4" y="1313.2" textLength="12.2" clip-path="url(#terminal-3189088113-line-53)">-</text><text class="terminal-3189088113-r5" x="36.6" y="1313.2" textLength="85.4" clip-path="url(#terminal-3189088113-line-53)">-github</text><text class="terminal-3189088113-r5" x="122" y="1313.2" textLength="134.2" clip-path="url(#term [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1337.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-54)">│</text><text class="terminal-3189088113-r5" x="24.4" y="1337.6" textLength="12.2" clip-path="url(#terminal-3189088113-line-54)">-</text><text class="terminal-3189088113-r5" x="36.6" y="1337.6" textLength="61" clip-path="url(#terminal-3189088113-line-54)">-help</text><text class="terminal-3189088113-r6" x="280.6" y="1337.6" textLength="24.4" clip-path="url(#termina [...]
+</text><text class="terminal-3189088113-r4" x="0" y="1362" textLength="1464" clip-path="url(#terminal-3189088113-line-55)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-3189088113-r2" x="1464" y="1362" textLength="12.2" clip-path="url(#terminal-3189088113-line-55)">
 </text>
     </g>
     </g>
diff --git a/images/breeze/output-commands-hash.txt b/images/breeze/output-commands-hash.txt
index e79557f189..43538474e1 100644
--- a/images/breeze/output-commands-hash.txt
+++ b/images/breeze/output-commands-hash.txt
@@ -2,4 +2,4 @@
 # This file is automatically generated by pre-commit. If you have a conflict with this file
 # Please do not solve it but run `breeze regenerate-command-images`.
 # This command should fix the conflict and regenerate help images that you have conflict with.
-eb0a4aef0e360063ff1bef04107752c7
+906b523aaaaed54525b239a97f3303c1
diff --git a/images/breeze/output-prepare-provider-documentation.svg b/images/breeze/output-prepare-provider-documentation.svg
index 49e1dd54b6..de6e1e4925 100644
--- a/images/breeze/output-prepare-provider-documentation.svg
+++ b/images/breeze/output-prepare-provider-documentation.svg
@@ -1,4 +1,4 @@
-<svg class="rich-terminal" viewBox="0 0 1482 684.4" xmlns="http://www.w3.org/2000/svg">
+<svg class="rich-terminal" viewBox="0 0 1482 708.8" xmlns="http://www.w3.org/2000/svg">
     <!-- Generated with Rich https://www.textualize.io -->
     <style>
 
@@ -19,145 +19,149 @@
         font-weight: 700;
     }
 
-    .terminal-2632715719-matrix {
+    .terminal-3621068628-matrix {
         font-family: Fira Code, monospace;
         font-size: 20px;
         line-height: 24.4px;
         font-variant-east-asian: full-width;
     }
 
-    .terminal-2632715719-title {
+    .terminal-3621068628-title {
         font-size: 18px;
         font-weight: bold;
         font-family: arial;
     }
 
-    .terminal-2632715719-r1 { fill: #c5c8c6;font-weight: bold }
-.terminal-2632715719-r2 { fill: #c5c8c6 }
-.terminal-2632715719-r3 { fill: #d0b344;font-weight: bold }
-.terminal-2632715719-r4 { fill: #868887 }
-.terminal-2632715719-r5 { fill: #68a0b3;font-weight: bold }
-.terminal-2632715719-r6 { fill: #98a84b;font-weight: bold }
-.terminal-2632715719-r7 { fill: #8d7b39 }
+    .terminal-3621068628-r1 { fill: #c5c8c6;font-weight: bold }
+.terminal-3621068628-r2 { fill: #c5c8c6 }
+.terminal-3621068628-r3 { fill: #d0b344;font-weight: bold }
+.terminal-3621068628-r4 { fill: #868887 }
+.terminal-3621068628-r5 { fill: #68a0b3;font-weight: bold }
+.terminal-3621068628-r6 { fill: #98a84b;font-weight: bold }
+.terminal-3621068628-r7 { fill: #8d7b39 }
     </style>
 
     <defs>
-    <clipPath id="terminal-2632715719-clip-terminal">
-      <rect x="0" y="0" width="1463.0" height="633.4" />
+    <clipPath id="terminal-3621068628-clip-terminal">
+      <rect x="0" y="0" width="1463.0" height="657.8" />
     </clipPath>
-    <clipPath id="terminal-2632715719-line-0">
+    <clipPath id="terminal-3621068628-line-0">
     <rect x="0" y="1.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-1">
+<clipPath id="terminal-3621068628-line-1">
     <rect x="0" y="25.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-2">
+<clipPath id="terminal-3621068628-line-2">
     <rect x="0" y="50.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-3">
+<clipPath id="terminal-3621068628-line-3">
     <rect x="0" y="74.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-4">
+<clipPath id="terminal-3621068628-line-4">
     <rect x="0" y="99.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-5">
+<clipPath id="terminal-3621068628-line-5">
     <rect x="0" y="123.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-6">
+<clipPath id="terminal-3621068628-line-6">
     <rect x="0" y="147.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-7">
+<clipPath id="terminal-3621068628-line-7">
     <rect x="0" y="172.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-8">
+<clipPath id="terminal-3621068628-line-8">
     <rect x="0" y="196.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-9">
+<clipPath id="terminal-3621068628-line-9">
     <rect x="0" y="221.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-10">
+<clipPath id="terminal-3621068628-line-10">
     <rect x="0" y="245.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-11">
+<clipPath id="terminal-3621068628-line-11">
     <rect x="0" y="269.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-12">
+<clipPath id="terminal-3621068628-line-12">
     <rect x="0" y="294.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-13">
+<clipPath id="terminal-3621068628-line-13">
     <rect x="0" y="318.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-14">
+<clipPath id="terminal-3621068628-line-14">
     <rect x="0" y="343.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-15">
+<clipPath id="terminal-3621068628-line-15">
     <rect x="0" y="367.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-16">
+<clipPath id="terminal-3621068628-line-16">
     <rect x="0" y="391.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-17">
+<clipPath id="terminal-3621068628-line-17">
     <rect x="0" y="416.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-18">
+<clipPath id="terminal-3621068628-line-18">
     <rect x="0" y="440.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-19">
+<clipPath id="terminal-3621068628-line-19">
     <rect x="0" y="465.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-20">
+<clipPath id="terminal-3621068628-line-20">
     <rect x="0" y="489.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-21">
+<clipPath id="terminal-3621068628-line-21">
     <rect x="0" y="513.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-22">
+<clipPath id="terminal-3621068628-line-22">
     <rect x="0" y="538.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-23">
+<clipPath id="terminal-3621068628-line-23">
     <rect x="0" y="562.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-2632715719-line-24">
+<clipPath id="terminal-3621068628-line-24">
     <rect x="0" y="587.1" width="1464" height="24.65"/>
             </clipPath>
+<clipPath id="terminal-3621068628-line-25">
+    <rect x="0" y="611.5" width="1464" height="24.65"/>
+            </clipPath>
     </defs>
 
-    <rect fill="#292929" stroke="rgba(255,255,255,0.35)" stroke-width="1" x="1" y="1" width="1480" height="682.4" rx="8"/><text class="terminal-2632715719-title" fill="#c5c8c6" text-anchor="middle" x="740" y="27">Command:&#160;prepare-provider-documentation</text>
+    <rect fill="#292929" stroke="rgba(255,255,255,0.35)" stroke-width="1" x="1" y="1" width="1480" height="706.8" rx="8"/><text class="terminal-3621068628-title" fill="#c5c8c6" text-anchor="middle" x="740" y="27">Command:&#160;prepare-provider-documentation</text>
             <g transform="translate(26,22)">
             <circle cx="0" cy="0" r="7" fill="#ff5f57"/>
             <circle cx="22" cy="0" r="7" fill="#febc2e"/>
             <circle cx="44" cy="0" r="7" fill="#28c840"/>
             </g>
         
-    <g transform="translate(9, 41)" clip-path="url(#terminal-2632715719-clip-terminal)">
+    <g transform="translate(9, 41)" clip-path="url(#terminal-3621068628-clip-terminal)">
     
-    <g class="terminal-2632715719-matrix">
-    <text class="terminal-2632715719-r2" x="1464" y="20" textLength="12.2" clip-path="url(#terminal-2632715719-line-0)">
-</text><text class="terminal-2632715719-r3" x="12.2" y="44.4" textLength="85.4" clip-path="url(#terminal-2632715719-line-1)">Usage:&#160;</text><text class="terminal-2632715719-r1" x="97.6" y="44.4" textLength="1342" clip-path="url(#terminal-2632715719-line-1)">breeze&#160;prepare-provider-documentation&#160;[OPTIONS]&#160;[airbyte&#160;|&#160;alibaba&#160;|&#160;amazon&#160;|&#160;apache.beam&#160;|&#160;apache.cassandra&#160;|</text><text class="terminal-2632715719-r2" x="1464" y="44.4 [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="68.8" textLength="1427.4" clip-path="url(#terminal-2632715719-line-2)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;apache.drill&#160;|&#160;apache.druid&#160;|&#160;apache.hdfs&#160;|&#160;apache.hive&#160;|&#1 [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="93.2" textLength="1415.2" clip-path="url(#terminal-2632715719-line-3)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;apache.livy&#160;|&#160;apache.pig&#160;|&#160;apache.pinot&#160;|&#160;apache.spark&#160;|&#16 [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="117.6" textLength="1390.8" clip-path="url(#terminal-2632715719-line-4)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;arangodb&#160;|&#160;asana&#160;|&#160;celery&#160;|&#160;cloudant&#160;|&#160;cncf.kubernetes [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="142" textLength="1366.4" clip-path="url(#terminal-2632715719-line-5)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;datadog&#160;|&#160;dbt.cloud&#160;|&#160;dingding&#160;|&#160;discord&#160;|&#160;docker&#160;| [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="166.4" textLength="1390.8" clip-path="url(#terminal-2632715719-line-6)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;exasol&#160;|&#160;facebook&#160;|&#160;ftp&#160;|&#160;github&#160;|&#160;google&#160;|&#160; [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="190.8" textLength="1268.8" clip-path="url(#terminal-2632715719-line-7)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;imap&#160;|&#160;influxdb&#160;|&#160;jdbc&#160;|&#160;jenkins&#160;|&#160;jira&#160;|&#160;mi [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="215.2" textLength="1378.6" clip-path="url(#terminal-2632715719-line-8)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;microsoft.mssql&#160;|&#160;microsoft.psrp&#160;|&#160;microsoft.winrm&#160;|&#160;mongo&#160; [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="239.6" textLength="1390.8" clip-path="url(#terminal-2632715719-line-9)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;neo4j&#160;|&#160;odbc&#160;|&#160;openfaas&#160;|&#160;opsgenie&#160;|&#160;oracle&#160;|&#16 [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="264" textLength="1354.2" clip-path="url(#terminal-2632715719-line-10)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;plexus&#160;|&#160;postgres&#160;|&#160;presto&#160;|&#160;qubole&#160;|&#160;redis&#160;|&#160 [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="288.4" textLength="1403" clip-path="url(#terminal-2632715719-line-11)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;segment&#160;|&#160;sendgrid&#160;|&#160;sftp&#160;|&#160;singularity&#160;|&#160;slack&#160;|& [...]
-</text><text class="terminal-2632715719-r1" x="12.2" y="312.8" textLength="1342" clip-path="url(#terminal-2632715719-line-12)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;ssh&#160;|&#160;tableau&#160;|&#160;telegram&#160;|&#160;trino&#160;|&#160;vertica&#160;|&#160; [...]
-</text><text class="terminal-2632715719-r2" x="1464" y="337.2" textLength="12.2" clip-path="url(#terminal-2632715719-line-13)">
-</text><text class="terminal-2632715719-r2" x="12.2" y="361.6" textLength="780.8" clip-path="url(#terminal-2632715719-line-14)">Prepare&#160;CHANGELOG,&#160;README&#160;and&#160;COMMITS&#160;information&#160;for&#160;providers.</text><text class="terminal-2632715719-r2" x="1464" y="361.6" textLength="12.2" clip-path="url(#terminal-2632715719-line-14)">
-</text><text class="terminal-2632715719-r2" x="1464" y="386" textLength="12.2" clip-path="url(#terminal-2632715719-line-15)">
-</text><text class="terminal-2632715719-r4" x="0" y="410.4" textLength="24.4" clip-path="url(#terminal-2632715719-line-16)">╭─</text><text class="terminal-2632715719-r4" x="24.4" y="410.4" textLength="1415.2" clip-path="url(#terminal-2632715719-line-16)">&#160;Provider&#160;documentation&#160;preparation&#160;flags&#160;──────────────────────────────────────────────────────────────────────────</text><text class="terminal-2632715719-r4" x="1439.6" y="410.4" textLength="24.4" clip-path="ur [...]
-</text><text class="terminal-2632715719-r4" x="0" y="434.8" textLength="12.2" clip-path="url(#terminal-2632715719-line-17)">│</text><text class="terminal-2632715719-r5" x="24.4" y="434.8" textLength="12.2" clip-path="url(#terminal-2632715719-line-17)">-</text><text class="terminal-2632715719-r5" x="36.6" y="434.8" textLength="73.2" clip-path="url(#terminal-2632715719-line-17)">-debug</text><text class="terminal-2632715719-r2" x="158.6" y="434.8" textLength="878.4" clip-path="url(#termina [...]
-</text><text class="terminal-2632715719-r4" x="0" y="459.2" textLength="1464" clip-path="url(#terminal-2632715719-line-18)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-2632715719-r2" x="1464" y="459.2" textLength="12.2" clip-path="url(#terminal-2632715719-line-18)">
-</text><text class="terminal-2632715719-r4" x="0" y="483.6" textLength="24.4" clip-path="url(#terminal-2632715719-line-19)">╭─</text><text class="terminal-2632715719-r4" x="24.4" y="483.6" textLength="1415.2" clip-path="url(#terminal-2632715719-line-19)">&#160;Options&#160;───────────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-2632715719-r4" x="1439.6" y="483.6" textLength="24.4" clip-path="url(#terminal-263 [...]
-</text><text class="terminal-2632715719-r4" x="0" y="508" textLength="12.2" clip-path="url(#terminal-2632715719-line-20)">│</text><text class="terminal-2632715719-r5" x="24.4" y="508" textLength="12.2" clip-path="url(#terminal-2632715719-line-20)">-</text><text class="terminal-2632715719-r5" x="36.6" y="508" textLength="97.6" clip-path="url(#terminal-2632715719-line-20)">-verbose</text><text class="terminal-2632715719-r6" x="280.6" y="508" textLength="24.4" clip-path="url(#terminal-26327 [...]
-</text><text class="terminal-2632715719-r4" x="0" y="532.4" textLength="12.2" clip-path="url(#terminal-2632715719-line-21)">│</text><text class="terminal-2632715719-r5" x="24.4" y="532.4" textLength="12.2" clip-path="url(#terminal-2632715719-line-21)">-</text><text class="terminal-2632715719-r5" x="36.6" y="532.4" textLength="48.8" clip-path="url(#terminal-2632715719-line-21)">-dry</text><text class="terminal-2632715719-r5" x="85.4" y="532.4" textLength="48.8" clip-path="url(#terminal-26 [...]
-</text><text class="terminal-2632715719-r4" x="0" y="556.8" textLength="12.2" clip-path="url(#terminal-2632715719-line-22)">│</text><text class="terminal-2632715719-r5" x="24.4" y="556.8" textLength="12.2" clip-path="url(#terminal-2632715719-line-22)">-</text><text class="terminal-2632715719-r5" x="36.6" y="556.8" textLength="85.4" clip-path="url(#terminal-2632715719-line-22)">-github</text><text class="terminal-2632715719-r5" x="122" y="556.8" textLength="134.2" clip-path="url(#terminal [...]
-</text><text class="terminal-2632715719-r4" x="0" y="581.2" textLength="12.2" clip-path="url(#terminal-2632715719-line-23)">│</text><text class="terminal-2632715719-r5" x="24.4" y="581.2" textLength="12.2" clip-path="url(#terminal-2632715719-line-23)">-</text><text class="terminal-2632715719-r5" x="36.6" y="581.2" textLength="85.4" clip-path="url(#terminal-2632715719-line-23)">-answer</text><text class="terminal-2632715719-r6" x="280.6" y="581.2" textLength="24.4" clip-path="url(#termina [...]
-</text><text class="terminal-2632715719-r4" x="0" y="605.6" textLength="12.2" clip-path="url(#terminal-2632715719-line-24)">│</text><text class="terminal-2632715719-r5" x="24.4" y="605.6" textLength="12.2" clip-path="url(#terminal-2632715719-line-24)">-</text><text class="terminal-2632715719-r5" x="36.6" y="605.6" textLength="61" clip-path="url(#terminal-2632715719-line-24)">-help</text><text class="terminal-2632715719-r6" x="280.6" y="605.6" textLength="24.4" clip-path="url(#terminal-26 [...]
-</text><text class="terminal-2632715719-r4" x="0" y="630" textLength="1464" clip-path="url(#terminal-2632715719-line-25)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-2632715719-r2" x="1464" y="630" textLength="12.2" clip-path="url(#terminal-2632715719-line-25)">
+    <g class="terminal-3621068628-matrix">
+    <text class="terminal-3621068628-r2" x="1464" y="20" textLength="12.2" clip-path="url(#terminal-3621068628-line-0)">
+</text><text class="terminal-3621068628-r3" x="12.2" y="44.4" textLength="85.4" clip-path="url(#terminal-3621068628-line-1)">Usage:&#160;</text><text class="terminal-3621068628-r1" x="97.6" y="44.4" textLength="1342" clip-path="url(#terminal-3621068628-line-1)">breeze&#160;prepare-provider-documentation&#160;[OPTIONS]&#160;[airbyte&#160;|&#160;alibaba&#160;|&#160;amazon&#160;|&#160;apache.beam&#160;|&#160;apache.cassandra&#160;|</text><text class="terminal-3621068628-r2" x="1464" y="44.4 [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="68.8" textLength="1427.4" clip-path="url(#terminal-3621068628-line-2)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;apache.drill&#160;|&#160;apache.druid&#160;|&#160;apache.hdfs&#160;|&#160;apache.hive&#160;|&#1 [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="93.2" textLength="1415.2" clip-path="url(#terminal-3621068628-line-3)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;apache.livy&#160;|&#160;apache.pig&#160;|&#160;apache.pinot&#160;|&#160;apache.spark&#160;|&#16 [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="117.6" textLength="1366.4" clip-path="url(#terminal-3621068628-line-4)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;arangodb&#160;|&#160;asana&#160;|&#160;celery&#160;|&#160;cloudant&#160;|&#160;cncf.kubernetes [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="142" textLength="1329.8" clip-path="url(#terminal-3621068628-line-5)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;databricks&#160;|&#160;datadog&#160;|&#160;dbt.cloud&#160;|&#160;dingding&#160;|&#160;discord&#1 [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="166.4" textLength="1354.2" clip-path="url(#terminal-3621068628-line-6)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;elasticsearch&#160;|&#160;exasol&#160;|&#160;facebook&#160;|&#160;ftp&#160;|&#160;github&#160; [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="190.8" textLength="1281" clip-path="url(#terminal-3621068628-line-7)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;hashicorp&#160;|&#160;http&#160;|&#160;imap&#160;|&#160;influxdb&#160;|&#160;jdbc&#160;|&#160;je [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="215.2" textLength="1403" clip-path="url(#terminal-3621068628-line-8)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;microsoft.azure&#160;|&#160;microsoft.mssql&#160;|&#160;microsoft.psrp&#160;|&#160;microsoft.win [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="239.6" textLength="1439.6" clip-path="url(#terminal-3621068628-line-9)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;mongo&#160;|&#160;mysql&#160;|&#160;neo4j&#160;|&#160;odbc&#160;|&#160;openfaas&#160;|&#160;op [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="264" textLength="1403" clip-path="url(#terminal-3621068628-line-10)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;papermill&#160;|&#160;plexus&#160;|&#160;postgres&#160;|&#160;presto&#160;|&#160;qubole&#160;|&#1 [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="288.4" textLength="1390.8" clip-path="url(#terminal-3621068628-line-11)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;samba&#160;|&#160;segment&#160;|&#160;sendgrid&#160;|&#160;sftp&#160;|&#160;singularity&#160; [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="312.8" textLength="1305.4" clip-path="url(#terminal-3621068628-line-12)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;sqlite&#160;|&#160;ssh&#160;|&#160;tableau&#160;|&#160;telegram&#160;|&#160;trino&#160;|&#160 [...]
+</text><text class="terminal-3621068628-r1" x="12.2" y="337.2" textLength="683.2" clip-path="url(#terminal-3621068628-line-13)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;zendesk]...</text><text class="terminal-3621068628-r2" x="1464" y="337.2" textLength="12.2" cl [...]
+</text><text class="terminal-3621068628-r2" x="1464" y="361.6" textLength="12.2" clip-path="url(#terminal-3621068628-line-14)">
+</text><text class="terminal-3621068628-r2" x="12.2" y="386" textLength="780.8" clip-path="url(#terminal-3621068628-line-15)">Prepare&#160;CHANGELOG,&#160;README&#160;and&#160;COMMITS&#160;information&#160;for&#160;providers.</text><text class="terminal-3621068628-r2" x="1464" y="386" textLength="12.2" clip-path="url(#terminal-3621068628-line-15)">
+</text><text class="terminal-3621068628-r2" x="1464" y="410.4" textLength="12.2" clip-path="url(#terminal-3621068628-line-16)">
+</text><text class="terminal-3621068628-r4" x="0" y="434.8" textLength="24.4" clip-path="url(#terminal-3621068628-line-17)">╭─</text><text class="terminal-3621068628-r4" x="24.4" y="434.8" textLength="1415.2" clip-path="url(#terminal-3621068628-line-17)">&#160;Provider&#160;documentation&#160;preparation&#160;flags&#160;──────────────────────────────────────────────────────────────────────────</text><text class="terminal-3621068628-r4" x="1439.6" y="434.8" textLength="24.4" clip-path="ur [...]
+</text><text class="terminal-3621068628-r4" x="0" y="459.2" textLength="12.2" clip-path="url(#terminal-3621068628-line-18)">│</text><text class="terminal-3621068628-r5" x="24.4" y="459.2" textLength="12.2" clip-path="url(#terminal-3621068628-line-18)">-</text><text class="terminal-3621068628-r5" x="36.6" y="459.2" textLength="73.2" clip-path="url(#terminal-3621068628-line-18)">-debug</text><text class="terminal-3621068628-r2" x="158.6" y="459.2" textLength="878.4" clip-path="url(#termina [...]
+</text><text class="terminal-3621068628-r4" x="0" y="483.6" textLength="1464" clip-path="url(#terminal-3621068628-line-19)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-3621068628-r2" x="1464" y="483.6" textLength="12.2" clip-path="url(#terminal-3621068628-line-19)">
+</text><text class="terminal-3621068628-r4" x="0" y="508" textLength="24.4" clip-path="url(#terminal-3621068628-line-20)">╭─</text><text class="terminal-3621068628-r4" x="24.4" y="508" textLength="1415.2" clip-path="url(#terminal-3621068628-line-20)">&#160;Options&#160;───────────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-3621068628-r4" x="1439.6" y="508" textLength="24.4" clip-path="url(#terminal-362106862 [...]
+</text><text class="terminal-3621068628-r4" x="0" y="532.4" textLength="12.2" clip-path="url(#terminal-3621068628-line-21)">│</text><text class="terminal-3621068628-r5" x="24.4" y="532.4" textLength="12.2" clip-path="url(#terminal-3621068628-line-21)">-</text><text class="terminal-3621068628-r5" x="36.6" y="532.4" textLength="97.6" clip-path="url(#terminal-3621068628-line-21)">-verbose</text><text class="terminal-3621068628-r6" x="280.6" y="532.4" textLength="24.4" clip-path="url(#termin [...]
+</text><text class="terminal-3621068628-r4" x="0" y="556.8" textLength="12.2" clip-path="url(#terminal-3621068628-line-22)">│</text><text class="terminal-3621068628-r5" x="24.4" y="556.8" textLength="12.2" clip-path="url(#terminal-3621068628-line-22)">-</text><text class="terminal-3621068628-r5" x="36.6" y="556.8" textLength="48.8" clip-path="url(#terminal-3621068628-line-22)">-dry</text><text class="terminal-3621068628-r5" x="85.4" y="556.8" textLength="48.8" clip-path="url(#terminal-36 [...]
+</text><text class="terminal-3621068628-r4" x="0" y="581.2" textLength="12.2" clip-path="url(#terminal-3621068628-line-23)">│</text><text class="terminal-3621068628-r5" x="24.4" y="581.2" textLength="12.2" clip-path="url(#terminal-3621068628-line-23)">-</text><text class="terminal-3621068628-r5" x="36.6" y="581.2" textLength="85.4" clip-path="url(#terminal-3621068628-line-23)">-github</text><text class="terminal-3621068628-r5" x="122" y="581.2" textLength="134.2" clip-path="url(#terminal [...]
+</text><text class="terminal-3621068628-r4" x="0" y="605.6" textLength="12.2" clip-path="url(#terminal-3621068628-line-24)">│</text><text class="terminal-3621068628-r5" x="24.4" y="605.6" textLength="12.2" clip-path="url(#terminal-3621068628-line-24)">-</text><text class="terminal-3621068628-r5" x="36.6" y="605.6" textLength="85.4" clip-path="url(#terminal-3621068628-line-24)">-answer</text><text class="terminal-3621068628-r6" x="280.6" y="605.6" textLength="24.4" clip-path="url(#termina [...]
+</text><text class="terminal-3621068628-r4" x="0" y="630" textLength="12.2" clip-path="url(#terminal-3621068628-line-25)">│</text><text class="terminal-3621068628-r5" x="24.4" y="630" textLength="12.2" clip-path="url(#terminal-3621068628-line-25)">-</text><text class="terminal-3621068628-r5" x="36.6" y="630" textLength="61" clip-path="url(#terminal-3621068628-line-25)">-help</text><text class="terminal-3621068628-r6" x="280.6" y="630" textLength="24.4" clip-path="url(#terminal-3621068628 [...]
+</text><text class="terminal-3621068628-r4" x="0" y="654.4" textLength="1464" clip-path="url(#terminal-3621068628-line-26)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-3621068628-r2" x="1464" y="654.4" textLength="12.2" clip-path="url(#terminal-3621068628-line-26)">
 </text>
     </g>
     </g>
diff --git a/images/breeze/output-prepare-provider-packages.svg b/images/breeze/output-prepare-provider-packages.svg
index 1da53c7d56..18a9bda713 100644
--- a/images/breeze/output-prepare-provider-packages.svg
+++ b/images/breeze/output-prepare-provider-packages.svg
@@ -19,153 +19,153 @@
         font-weight: 700;
     }
 
-    .terminal-3636495061-matrix {
+    .terminal-3886318936-matrix {
         font-family: Fira Code, monospace;
         font-size: 20px;
         line-height: 24.4px;
         font-variant-east-asian: full-width;
     }
 
-    .terminal-3636495061-title {
+    .terminal-3886318936-title {
         font-size: 18px;
         font-weight: bold;
         font-family: arial;
     }
 
-    .terminal-3636495061-r1 { fill: #c5c8c6;font-weight: bold }
-.terminal-3636495061-r2 { fill: #c5c8c6 }
-.terminal-3636495061-r3 { fill: #d0b344;font-weight: bold }
-.terminal-3636495061-r4 { fill: #868887 }
-.terminal-3636495061-r5 { fill: #68a0b3;font-weight: bold }
-.terminal-3636495061-r6 { fill: #8d7b39 }
-.terminal-3636495061-r7 { fill: #98a84b;font-weight: bold }
+    .terminal-3886318936-r1 { fill: #c5c8c6;font-weight: bold }
+.terminal-3886318936-r2 { fill: #c5c8c6 }
+.terminal-3886318936-r3 { fill: #d0b344;font-weight: bold }
+.terminal-3886318936-r4 { fill: #868887 }
+.terminal-3886318936-r5 { fill: #68a0b3;font-weight: bold }
+.terminal-3886318936-r6 { fill: #8d7b39 }
+.terminal-3886318936-r7 { fill: #98a84b;font-weight: bold }
     </style>
 
     <defs>
-    <clipPath id="terminal-3636495061-clip-terminal">
+    <clipPath id="terminal-3886318936-clip-terminal">
       <rect x="0" y="0" width="1463.0" height="682.1999999999999" />
     </clipPath>
-    <clipPath id="terminal-3636495061-line-0">
+    <clipPath id="terminal-3886318936-line-0">
     <rect x="0" y="1.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-1">
+<clipPath id="terminal-3886318936-line-1">
     <rect x="0" y="25.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-2">
+<clipPath id="terminal-3886318936-line-2">
     <rect x="0" y="50.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-3">
+<clipPath id="terminal-3886318936-line-3">
     <rect x="0" y="74.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-4">
+<clipPath id="terminal-3886318936-line-4">
     <rect x="0" y="99.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-5">
+<clipPath id="terminal-3886318936-line-5">
     <rect x="0" y="123.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-6">
+<clipPath id="terminal-3886318936-line-6">
     <rect x="0" y="147.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-7">
+<clipPath id="terminal-3886318936-line-7">
     <rect x="0" y="172.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-8">
+<clipPath id="terminal-3886318936-line-8">
     <rect x="0" y="196.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-9">
+<clipPath id="terminal-3886318936-line-9">
     <rect x="0" y="221.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-10">
+<clipPath id="terminal-3886318936-line-10">
     <rect x="0" y="245.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-11">
+<clipPath id="terminal-3886318936-line-11">
     <rect x="0" y="269.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-12">
+<clipPath id="terminal-3886318936-line-12">
     <rect x="0" y="294.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-13">
+<clipPath id="terminal-3886318936-line-13">
     <rect x="0" y="318.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-14">
+<clipPath id="terminal-3886318936-line-14">
     <rect x="0" y="343.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-15">
+<clipPath id="terminal-3886318936-line-15">
     <rect x="0" y="367.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-16">
+<clipPath id="terminal-3886318936-line-16">
     <rect x="0" y="391.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-17">
+<clipPath id="terminal-3886318936-line-17">
     <rect x="0" y="416.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-18">
+<clipPath id="terminal-3886318936-line-18">
     <rect x="0" y="440.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-19">
+<clipPath id="terminal-3886318936-line-19">
     <rect x="0" y="465.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-20">
+<clipPath id="terminal-3886318936-line-20">
     <rect x="0" y="489.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-21">
+<clipPath id="terminal-3886318936-line-21">
     <rect x="0" y="513.9" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-22">
+<clipPath id="terminal-3886318936-line-22">
     <rect x="0" y="538.3" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-23">
+<clipPath id="terminal-3886318936-line-23">
     <rect x="0" y="562.7" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-24">
+<clipPath id="terminal-3886318936-line-24">
     <rect x="0" y="587.1" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-25">
+<clipPath id="terminal-3886318936-line-25">
     <rect x="0" y="611.5" width="1464" height="24.65"/>
             </clipPath>
-<clipPath id="terminal-3636495061-line-26">
+<clipPath id="terminal-3886318936-line-26">
     <rect x="0" y="635.9" width="1464" height="24.65"/>
             </clipPath>
     </defs>
 
-    <rect fill="#292929" stroke="rgba(255,255,255,0.35)" stroke-width="1" x="1" y="1" width="1480" height="731.2" rx="8"/><text class="terminal-3636495061-title" fill="#c5c8c6" text-anchor="middle" x="740" y="27">Command:&#160;prepare-provider-packages</text>
+    <rect fill="#292929" stroke="rgba(255,255,255,0.35)" stroke-width="1" x="1" y="1" width="1480" height="731.2" rx="8"/><text class="terminal-3886318936-title" fill="#c5c8c6" text-anchor="middle" x="740" y="27">Command:&#160;prepare-provider-packages</text>
             <g transform="translate(26,22)">
             <circle cx="0" cy="0" r="7" fill="#ff5f57"/>
             <circle cx="22" cy="0" r="7" fill="#febc2e"/>
             <circle cx="44" cy="0" r="7" fill="#28c840"/>
             </g>
         
-    <g transform="translate(9, 41)" clip-path="url(#terminal-3636495061-clip-terminal)">
+    <g transform="translate(9, 41)" clip-path="url(#terminal-3886318936-clip-terminal)">
     
-    <g class="terminal-3636495061-matrix">
-    <text class="terminal-3636495061-r2" x="1464" y="20" textLength="12.2" clip-path="url(#terminal-3636495061-line-0)">
-</text><text class="terminal-3636495061-r3" x="12.2" y="44.4" textLength="85.4" clip-path="url(#terminal-3636495061-line-1)">Usage:&#160;</text><text class="terminal-3636495061-r1" x="97.6" y="44.4" textLength="1281" clip-path="url(#terminal-3636495061-line-1)">breeze&#160;prepare-provider-packages&#160;[OPTIONS]&#160;[airbyte&#160;|&#160;alibaba&#160;|&#160;amazon&#160;|&#160;apache.beam&#160;|&#160;apache.cassandra&#160;|</text><text class="terminal-3636495061-r2" x="1464" y="44.4" tex [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="68.8" textLength="1366.4" clip-path="url(#terminal-3636495061-line-2)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;apache.drill&#160;|&#160;apache.druid&#160;|&#160;apache.hdfs&#160;|&#160;apache.hive&#160;|&#160;apache.kylin&#160;|</text>< [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="93.2" textLength="1354.2" clip-path="url(#terminal-3636495061-line-3)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;apache.livy&#160;|&#160;apache.pig&#160;|&#160;apache.pinot&#160;|&#160;apache.spark&#160;|&#160;apache.sqoop&#160;|</text><t [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="117.6" textLength="1427.4" clip-path="url(#terminal-3636495061-line-4)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;arangodb&#160;|&#160;asana&#160;|&#160;celery&#160;|&#160;cloudant&#160;|&#160;cncf.kubernetes&#160;|&#160;databricks&#160;| [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="142" textLength="1427.4" clip-path="url(#terminal-3636495061-line-5)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;|&#160;dbt.cloud&#160;|&#160;dingding&#160;|&#160;discord&#160;|&#160;docker&#160;|&#160;elasticsearch&#160;|&#160;exasol&#160 [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="166.4" textLength="1415.2" clip-path="url(#terminal-3636495061-line-6)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;|&#160;ftp&#160;|&#160;github&#160;|&#160;google&#160;|&#160;grpc&#160;|&#160;hashicorp&#160;|&#160;http&#160;|&#160;imap&#1 [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="190.8" textLength="1329.8" clip-path="url(#terminal-3636495061-line-7)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;jenkins&#160;|&#160;jira&#160;|&#160;microsoft.azure&#160;|&#160;microsoft.mssql&#160;|&#160;microsoft.psrp&#160;|</text><te [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="215.2" textLength="1427.4" clip-path="url(#terminal-3636495061-line-8)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;microsoft.winrm&#160;|&#160;mongo&#160;|&#160;mysql&#160;|&#160;neo4j&#160;|&#160;odbc&#160;|&#160;openfaas&#160;|&#160;opsg [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="239.6" textLength="1354.2" clip-path="url(#terminal-3636495061-line-9)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;|&#160;pagerduty&#160;|&#160;papermill&#160;|&#160;plexus&#160;|&#160;postgres&#160;|&#160;presto&#160;|&#160;qubole&#160;|& [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="264" textLength="1342" clip-path="url(#terminal-3636495061-line-10)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;salesforce&#160;|&#160;samba&#160;|&#160;segment&#160;|&#160;sendgrid&#160;|&#160;sftp&#160;|&#160;singularity&#160;|&#160;slac [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="288.4" textLength="1390.8" clip-path="url(#terminal-3636495061-line-11)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;snowflake&#160;|&#160;sqlite&#160;|&#160;ssh&#160;|&#160;tableau&#160;|&#160;telegram&#160;|&#160;trino&#160;|&#160;vertica [...]
-</text><text class="terminal-3636495061-r1" x="12.2" y="312.8" textLength="622.2" clip-path="url(#terminal-3636495061-line-12)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;zendesk]...</text><text class="terminal-3636495061-r2" x="1464" y="312.8" textLength="12.2" clip-path="url(#terminal-3636495 [...]
-</text><text class="terminal-3636495061-r2" x="1464" y="337.2" textLength="12.2" clip-path="url(#terminal-3636495061-line-13)">
-</text><text class="terminal-3636495061-r2" x="12.2" y="361.6" textLength="585.6" clip-path="url(#terminal-3636495061-line-14)">Prepare&#160;sdist/whl&#160;packages&#160;of&#160;Airflow&#160;Providers.</text><text class="terminal-3636495061-r2" x="1464" y="361.6" textLength="12.2" clip-path="url(#terminal-3636495061-line-14)">
-</text><text class="terminal-3636495061-r2" x="1464" y="386" textLength="12.2" clip-path="url(#terminal-3636495061-line-15)">
-</text><text class="terminal-3636495061-r4" x="0" y="410.4" textLength="24.4" clip-path="url(#terminal-3636495061-line-16)">╭─</text><text class="terminal-3636495061-r4" x="24.4" y="410.4" textLength="1415.2" clip-path="url(#terminal-3636495061-line-16)">&#160;Package&#160;flags&#160;─────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-3636495061-r4" x="1439.6" y="410.4" textLength="24.4" clip-path="url(#termina [...]
-</text><text class="terminal-3636495061-r4" x="0" y="434.8" textLength="12.2" clip-path="url(#terminal-3636495061-line-17)">│</text><text class="terminal-3636495061-r5" x="24.4" y="434.8" textLength="12.2" clip-path="url(#terminal-3636495061-line-17)">-</text><text class="terminal-3636495061-r5" x="36.6" y="434.8" textLength="97.6" clip-path="url(#terminal-3636495061-line-17)">-package</text><text class="terminal-3636495061-r5" x="134.2" y="434.8" textLength="85.4" clip-path="url(#termin [...]
-</text><text class="terminal-3636495061-r4" x="0" y="459.2" textLength="12.2" clip-path="url(#terminal-3636495061-line-18)">│</text><text class="terminal-3636495061-r5" x="24.4" y="459.2" textLength="12.2" clip-path="url(#terminal-3636495061-line-18)">-</text><text class="terminal-3636495061-r5" x="36.6" y="459.2" textLength="97.6" clip-path="url(#terminal-3636495061-line-18)">-version</text><text class="terminal-3636495061-r5" x="134.2" y="459.2" textLength="195.2" clip-path="url(#termi [...]
-</text><text class="terminal-3636495061-r4" x="0" y="483.6" textLength="12.2" clip-path="url(#terminal-3636495061-line-19)">│</text><text class="terminal-3636495061-r5" x="24.4" y="483.6" textLength="12.2" clip-path="url(#terminal-3636495061-line-19)">-</text><text class="terminal-3636495061-r5" x="36.6" y="483.6" textLength="97.6" clip-path="url(#terminal-3636495061-line-19)">-package</text><text class="terminal-3636495061-r5" x="134.2" y="483.6" textLength="122" clip-path="url(#termina [...]
-</text><text class="terminal-3636495061-r4" x="0" y="508" textLength="12.2" clip-path="url(#terminal-3636495061-line-20)">│</text><text class="terminal-3636495061-r5" x="24.4" y="508" textLength="12.2" clip-path="url(#terminal-3636495061-line-20)">-</text><text class="terminal-3636495061-r5" x="36.6" y="508" textLength="73.2" clip-path="url(#terminal-3636495061-line-20)">-debug</text><text class="terminal-3636495061-r2" x="378.2" y="508" textLength="878.4" clip-path="url(#terminal-363649 [...]
-</text><text class="terminal-3636495061-r4" x="0" y="532.4" textLength="1464" clip-path="url(#terminal-3636495061-line-21)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-3636495061-r2" x="1464" y="532.4" textLength="12.2" clip-path="url(#terminal-3636495061-line-21)">
-</text><text class="terminal-3636495061-r4" x="0" y="556.8" textLength="24.4" clip-path="url(#terminal-3636495061-line-22)">╭─</text><text class="terminal-3636495061-r4" x="24.4" y="556.8" textLength="1415.2" clip-path="url(#terminal-3636495061-line-22)">&#160;Options&#160;───────────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-3636495061-r4" x="1439.6" y="556.8" textLength="24.4" clip-path="url(#terminal-363 [...]
-</text><text class="terminal-3636495061-r4" x="0" y="581.2" textLength="12.2" clip-path="url(#terminal-3636495061-line-23)">│</text><text class="terminal-3636495061-r5" x="24.4" y="581.2" textLength="12.2" clip-path="url(#terminal-3636495061-line-23)">-</text><text class="terminal-3636495061-r5" x="36.6" y="581.2" textLength="97.6" clip-path="url(#terminal-3636495061-line-23)">-verbose</text><text class="terminal-3636495061-r7" x="280.6" y="581.2" textLength="24.4" clip-path="url(#termin [...]
-</text><text class="terminal-3636495061-r4" x="0" y="605.6" textLength="12.2" clip-path="url(#terminal-3636495061-line-24)">│</text><text class="terminal-3636495061-r5" x="24.4" y="605.6" textLength="12.2" clip-path="url(#terminal-3636495061-line-24)">-</text><text class="terminal-3636495061-r5" x="36.6" y="605.6" textLength="48.8" clip-path="url(#terminal-3636495061-line-24)">-dry</text><text class="terminal-3636495061-r5" x="85.4" y="605.6" textLength="48.8" clip-path="url(#terminal-36 [...]
-</text><text class="terminal-3636495061-r4" x="0" y="630" textLength="12.2" clip-path="url(#terminal-3636495061-line-25)">│</text><text class="terminal-3636495061-r5" x="24.4" y="630" textLength="12.2" clip-path="url(#terminal-3636495061-line-25)">-</text><text class="terminal-3636495061-r5" x="36.6" y="630" textLength="85.4" clip-path="url(#terminal-3636495061-line-25)">-github</text><text class="terminal-3636495061-r5" x="122" y="630" textLength="134.2" clip-path="url(#terminal-3636495 [...]
-</text><text class="terminal-3636495061-r4" x="0" y="654.4" textLength="12.2" clip-path="url(#terminal-3636495061-line-26)">│</text><text class="terminal-3636495061-r5" x="24.4" y="654.4" textLength="12.2" clip-path="url(#terminal-3636495061-line-26)">-</text><text class="terminal-3636495061-r5" x="36.6" y="654.4" textLength="61" clip-path="url(#terminal-3636495061-line-26)">-help</text><text class="terminal-3636495061-r7" x="280.6" y="654.4" textLength="24.4" clip-path="url(#terminal-36 [...]
-</text><text class="terminal-3636495061-r4" x="0" y="678.8" textLength="1464" clip-path="url(#terminal-3636495061-line-27)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-3636495061-r2" x="1464" y="678.8" textLength="12.2" clip-path="url(#terminal-3636495061-line-27)">
+    <g class="terminal-3886318936-matrix">
+    <text class="terminal-3886318936-r2" x="1464" y="20" textLength="12.2" clip-path="url(#terminal-3886318936-line-0)">
+</text><text class="terminal-3886318936-r3" x="12.2" y="44.4" textLength="85.4" clip-path="url(#terminal-3886318936-line-1)">Usage:&#160;</text><text class="terminal-3886318936-r1" x="97.6" y="44.4" textLength="1281" clip-path="url(#terminal-3886318936-line-1)">breeze&#160;prepare-provider-packages&#160;[OPTIONS]&#160;[airbyte&#160;|&#160;alibaba&#160;|&#160;amazon&#160;|&#160;apache.beam&#160;|&#160;apache.cassandra&#160;|</text><text class="terminal-3886318936-r2" x="1464" y="44.4" tex [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="68.8" textLength="1366.4" clip-path="url(#terminal-3886318936-line-2)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;apache.drill&#160;|&#160;apache.druid&#160;|&#160;apache.hdfs&#160;|&#160;apache.hive&#160;|&#160;apache.kylin&#160;|</text>< [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="93.2" textLength="1354.2" clip-path="url(#terminal-3886318936-line-3)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;apache.livy&#160;|&#160;apache.pig&#160;|&#160;apache.pinot&#160;|&#160;apache.spark&#160;|&#160;apache.sqoop&#160;|</text><t [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="117.6" textLength="1439.6" clip-path="url(#terminal-3886318936-line-4)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;arangodb&#160;|&#160;asana&#160;|&#160;celery&#160;|&#160;cloudant&#160;|&#160;cncf.kubernetes&#160;|&#160;core.sql&#160;|&# [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="142" textLength="1439.6" clip-path="url(#terminal-3886318936-line-5)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;|&#160;datadog&#160;|&#160;dbt.cloud&#160;|&#160;dingding&#160;|&#160;discord&#160;|&#160;docker&#160;|&#160;elasticsearch&#16 [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="166.4" textLength="1439.6" clip-path="url(#terminal-3886318936-line-6)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;facebook&#160;|&#160;ftp&#160;|&#160;github&#160;|&#160;google&#160;|&#160;grpc&#160;|&#160;hashicorp&#160;|&#160;http&#160; [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="190.8" textLength="1415.2" clip-path="url(#terminal-3886318936-line-7)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;jdbc&#160;|&#160;jenkins&#160;|&#160;jira&#160;|&#160;microsoft.azure&#160;|&#160;microsoft.mssql&#160;|&#160;microsoft.psrp [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="215.2" textLength="1427.4" clip-path="url(#terminal-3886318936-line-8)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;microsoft.winrm&#160;|&#160;mongo&#160;|&#160;mysql&#160;|&#160;neo4j&#160;|&#160;odbc&#160;|&#160;openfaas&#160;|&#160;opsg [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="239.6" textLength="1354.2" clip-path="url(#terminal-3886318936-line-9)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;|&#160;pagerduty&#160;|&#160;papermill&#160;|&#160;plexus&#160;|&#160;postgres&#160;|&#160;presto&#160;|&#160;qubole&#160;|& [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="264" textLength="1342" clip-path="url(#terminal-3886318936-line-10)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;salesforce&#160;|&#160;samba&#160;|&#160;segment&#160;|&#160;sendgrid&#160;|&#160;sftp&#160;|&#160;singularity&#160;|&#160;slac [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="288.4" textLength="1390.8" clip-path="url(#terminal-3886318936-line-11)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;snowflake&#160;|&#160;sqlite&#160;|&#160;ssh&#160;|&#160;tableau&#160;|&#160;telegram&#160;|&#160;trino&#160;|&#160;vertica [...]
+</text><text class="terminal-3886318936-r1" x="12.2" y="312.8" textLength="622.2" clip-path="url(#terminal-3886318936-line-12)">&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;&#160;zendesk]...</text><text class="terminal-3886318936-r2" x="1464" y="312.8" textLength="12.2" clip-path="url(#terminal-3886318 [...]
+</text><text class="terminal-3886318936-r2" x="1464" y="337.2" textLength="12.2" clip-path="url(#terminal-3886318936-line-13)">
+</text><text class="terminal-3886318936-r2" x="12.2" y="361.6" textLength="585.6" clip-path="url(#terminal-3886318936-line-14)">Prepare&#160;sdist/whl&#160;packages&#160;of&#160;Airflow&#160;Providers.</text><text class="terminal-3886318936-r2" x="1464" y="361.6" textLength="12.2" clip-path="url(#terminal-3886318936-line-14)">
+</text><text class="terminal-3886318936-r2" x="1464" y="386" textLength="12.2" clip-path="url(#terminal-3886318936-line-15)">
+</text><text class="terminal-3886318936-r4" x="0" y="410.4" textLength="24.4" clip-path="url(#terminal-3886318936-line-16)">╭─</text><text class="terminal-3886318936-r4" x="24.4" y="410.4" textLength="1415.2" clip-path="url(#terminal-3886318936-line-16)">&#160;Package&#160;flags&#160;─────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-3886318936-r4" x="1439.6" y="410.4" textLength="24.4" clip-path="url(#termina [...]
+</text><text class="terminal-3886318936-r4" x="0" y="434.8" textLength="12.2" clip-path="url(#terminal-3886318936-line-17)">│</text><text class="terminal-3886318936-r5" x="24.4" y="434.8" textLength="12.2" clip-path="url(#terminal-3886318936-line-17)">-</text><text class="terminal-3886318936-r5" x="36.6" y="434.8" textLength="97.6" clip-path="url(#terminal-3886318936-line-17)">-package</text><text class="terminal-3886318936-r5" x="134.2" y="434.8" textLength="85.4" clip-path="url(#termin [...]
+</text><text class="terminal-3886318936-r4" x="0" y="459.2" textLength="12.2" clip-path="url(#terminal-3886318936-line-18)">│</text><text class="terminal-3886318936-r5" x="24.4" y="459.2" textLength="12.2" clip-path="url(#terminal-3886318936-line-18)">-</text><text class="terminal-3886318936-r5" x="36.6" y="459.2" textLength="97.6" clip-path="url(#terminal-3886318936-line-18)">-version</text><text class="terminal-3886318936-r5" x="134.2" y="459.2" textLength="195.2" clip-path="url(#termi [...]
+</text><text class="terminal-3886318936-r4" x="0" y="483.6" textLength="12.2" clip-path="url(#terminal-3886318936-line-19)">│</text><text class="terminal-3886318936-r5" x="24.4" y="483.6" textLength="12.2" clip-path="url(#terminal-3886318936-line-19)">-</text><text class="terminal-3886318936-r5" x="36.6" y="483.6" textLength="97.6" clip-path="url(#terminal-3886318936-line-19)">-package</text><text class="terminal-3886318936-r5" x="134.2" y="483.6" textLength="122" clip-path="url(#termina [...]
+</text><text class="terminal-3886318936-r4" x="0" y="508" textLength="12.2" clip-path="url(#terminal-3886318936-line-20)">│</text><text class="terminal-3886318936-r5" x="24.4" y="508" textLength="12.2" clip-path="url(#terminal-3886318936-line-20)">-</text><text class="terminal-3886318936-r5" x="36.6" y="508" textLength="73.2" clip-path="url(#terminal-3886318936-line-20)">-debug</text><text class="terminal-3886318936-r2" x="378.2" y="508" textLength="878.4" clip-path="url(#terminal-388631 [...]
+</text><text class="terminal-3886318936-r4" x="0" y="532.4" textLength="1464" clip-path="url(#terminal-3886318936-line-21)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-3886318936-r2" x="1464" y="532.4" textLength="12.2" clip-path="url(#terminal-3886318936-line-21)">
+</text><text class="terminal-3886318936-r4" x="0" y="556.8" textLength="24.4" clip-path="url(#terminal-3886318936-line-22)">╭─</text><text class="terminal-3886318936-r4" x="24.4" y="556.8" textLength="1415.2" clip-path="url(#terminal-3886318936-line-22)">&#160;Options&#160;───────────────────────────────────────────────────────────────────────────────────────────────────────────</text><text class="terminal-3886318936-r4" x="1439.6" y="556.8" textLength="24.4" clip-path="url(#terminal-388 [...]
+</text><text class="terminal-3886318936-r4" x="0" y="581.2" textLength="12.2" clip-path="url(#terminal-3886318936-line-23)">│</text><text class="terminal-3886318936-r5" x="24.4" y="581.2" textLength="12.2" clip-path="url(#terminal-3886318936-line-23)">-</text><text class="terminal-3886318936-r5" x="36.6" y="581.2" textLength="97.6" clip-path="url(#terminal-3886318936-line-23)">-verbose</text><text class="terminal-3886318936-r7" x="280.6" y="581.2" textLength="24.4" clip-path="url(#termin [...]
+</text><text class="terminal-3886318936-r4" x="0" y="605.6" textLength="12.2" clip-path="url(#terminal-3886318936-line-24)">│</text><text class="terminal-3886318936-r5" x="24.4" y="605.6" textLength="12.2" clip-path="url(#terminal-3886318936-line-24)">-</text><text class="terminal-3886318936-r5" x="36.6" y="605.6" textLength="48.8" clip-path="url(#terminal-3886318936-line-24)">-dry</text><text class="terminal-3886318936-r5" x="85.4" y="605.6" textLength="48.8" clip-path="url(#terminal-38 [...]
+</text><text class="terminal-3886318936-r4" x="0" y="630" textLength="12.2" clip-path="url(#terminal-3886318936-line-25)">│</text><text class="terminal-3886318936-r5" x="24.4" y="630" textLength="12.2" clip-path="url(#terminal-3886318936-line-25)">-</text><text class="terminal-3886318936-r5" x="36.6" y="630" textLength="85.4" clip-path="url(#terminal-3886318936-line-25)">-github</text><text class="terminal-3886318936-r5" x="122" y="630" textLength="134.2" clip-path="url(#terminal-3886318 [...]
+</text><text class="terminal-3886318936-r4" x="0" y="654.4" textLength="12.2" clip-path="url(#terminal-3886318936-line-26)">│</text><text class="terminal-3886318936-r5" x="24.4" y="654.4" textLength="12.2" clip-path="url(#terminal-3886318936-line-26)">-</text><text class="terminal-3886318936-r5" x="36.6" y="654.4" textLength="61" clip-path="url(#terminal-3886318936-line-26)">-help</text><text class="terminal-3886318936-r7" x="280.6" y="654.4" textLength="24.4" clip-path="url(#terminal-38 [...]
+</text><text class="terminal-3886318936-r4" x="0" y="678.8" textLength="1464" clip-path="url(#terminal-3886318936-line-27)">╰──────────────────────────────────────────────────────────────────────────────────────────────────────────────────────╯</text><text class="terminal-3886318936-r2" x="1464" y="678.8" textLength="12.2" clip-path="url(#terminal-3886318936-line-27)">
 </text>
     </g>
     </g>
diff --git a/setup.py b/setup.py
index b4baa640b4..996c66c736 100644
--- a/setup.py
+++ b/setup.py
@@ -683,6 +683,7 @@ PROVIDERS_REQUIREMENTS: Dict[str, List[str]] = {
     'celery': celery,
     'cloudant': cloudant,
     'cncf.kubernetes': kubernetes,
+    'core.sql': [],
     'databricks': databricks,
     'datadog': datadog,
     'dbt.cloud': http_provider,
diff --git a/tests/providers/core/__init__.py b/tests/providers/core/__init__.py
new file mode 100644
index 0000000000..13a83393a9
--- /dev/null
+++ b/tests/providers/core/__init__.py
@@ -0,0 +1,16 @@
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
diff --git a/tests/providers/core/sql/__init__.py b/tests/providers/core/sql/__init__.py
new file mode 100644
index 0000000000..13a83393a9
--- /dev/null
+++ b/tests/providers/core/sql/__init__.py
@@ -0,0 +1,16 @@
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
diff --git a/tests/providers/core/sql/operators/__init__.py b/tests/providers/core/sql/operators/__init__.py
new file mode 100644
index 0000000000..13a83393a9
--- /dev/null
+++ b/tests/providers/core/sql/operators/__init__.py
@@ -0,0 +1,16 @@
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
diff --git a/tests/providers/core/sql/operators/test_sql.py b/tests/providers/core/sql/operators/test_sql.py
new file mode 100644
index 0000000000..d39e2d9fae
--- /dev/null
+++ b/tests/providers/core/sql/operators/test_sql.py
@@ -0,0 +1,114 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+
+import pytest
+
+from airflow.exceptions import AirflowException
+from airflow.providers.core.sql.operators.sql import SQLColumnCheckOperator, SQLTableCheckOperator
+
+
+class MockHook:
+    def get_first(self):
+        return
+
+
+def _get_mock_db_hook():
+    return MockHook()
+
+
+class TestColumnCheckOperator:
+
+    valid_column_mapping = {
+        "X": {
+            "null_check": {"equal_to": 0},
+            "distinct_check": {"equal_to": 10, "tolerance": 0.1},
+            "unique_check": {"geq_to": 10},
+            "min": {"leq_to": 1},
+            "max": {"less_than": 20, "greater_than": 10},
+        }
+    }
+
+    invalid_column_mapping = {"Y": {"invalid_check_name": {"expectation": 5}}}
+
+    def _construct_operator(self, monkeypatch, column_mapping, return_vals):
+        def get_first_return(*arg):
+            return return_vals
+
+        operator = SQLColumnCheckOperator(
+            task_id="test_task", table="test_table", column_mapping=column_mapping
+        )
+        monkeypatch.setattr(operator, "get_db_hook", _get_mock_db_hook)
+        monkeypatch.setattr(MockHook, "get_first", get_first_return)
+        return operator
+
+    def test_check_not_in_column_checks(self, monkeypatch):
+        with pytest.raises(AirflowException, match="Invalid column check: invalid_check_name."):
+            self._construct_operator(monkeypatch, self.invalid_column_mapping, ())
+
+    def test_pass_all_checks_exact_check(self, monkeypatch):
+        operator = self._construct_operator(monkeypatch, self.valid_column_mapping, (0, 10, 10, 1, 19))
+        operator.execute()
+
+    def test_max_less_than_fails_check(self, monkeypatch):
+        with pytest.raises(AirflowException):
+            operator = self._construct_operator(monkeypatch, self.valid_column_mapping, (0, 10, 10, 1, 21))
+            operator.execute()
+            assert operator.column_mapping["X"]["max"]["success"] is False
+
+    def test_max_greater_than_fails_check(self, monkeypatch):
+        with pytest.raises(AirflowException):
+            operator = self._construct_operator(monkeypatch, self.valid_column_mapping, (0, 10, 10, 1, 9))
+            operator.execute()
+            assert operator.column_mapping["X"]["max"]["success"] is False
+
+    def test_pass_all_checks_inexact_check(self, monkeypatch):
+        operator = self._construct_operator(monkeypatch, self.valid_column_mapping, (0, 9, 12, 0, 15))
+        operator.execute()
+
+    def test_fail_all_checks_check(self, monkeypatch):
+        operator = operator = self._construct_operator(
+            monkeypatch, self.valid_column_mapping, (1, 12, 11, -1, 20)
+        )
+        with pytest.raises(AirflowException):
+            operator.execute()
+
+
+class TestTableCheckOperator:
+
+    checks = {
+        "row_count_check": {"check_statement": "COUNT(*) == 1000"},
+        "column_sum_check": {"check_statement": "col_a + col_b < col_c"},
+    }
+
+    def _construct_operator(self, monkeypatch, checks, return_vals):
+        def get_first_return(*arg):
+            return return_vals
+
+        operator = SQLTableCheckOperator(task_id="test_task", table="test_table", checks=checks)
+        monkeypatch.setattr(operator, "get_db_hook", _get_mock_db_hook)
+        monkeypatch.setattr(MockHook, "get_first", get_first_return)
+        return operator
+
+    def test_pass_all_checks_check(self, monkeypatch):
+        operator = self._construct_operator(monkeypatch, self.checks, ('1', 'y', 'true'))
+        operator.execute()
+
+    def test_fail_all_checks_check(self, monkeypatch):
+        operator = self._construct_operator(monkeypatch, self.checks, ('0', 'n', 'false'))
+        with pytest.raises(AirflowException):
+            operator.execute()