Skip to content

Commit 037adbf

Browse files
committed
Remove email conigs, operator and callbacks
1 parent dd2252d commit 037adbf

File tree

33 files changed

+99
-1558
lines changed

33 files changed

+99
-1558
lines changed

airflow/api_connexion/openapi/v1.yaml

Lines changed: 0 additions & 10 deletions
Original file line numberDiff line numberDiff line change
@@ -2488,22 +2488,12 @@ paths:
24882488
options:
24892489
- key: dags_folder
24902490
value: /home/user/my-dags-folder
2491-
2492-
- name: smtp
2493-
options:
2494-
- key: smtp_host
2495-
value: localhost
2496-
- key: smtp_mail_from
2497-
24982491
text/plain:
24992492
schema:
25002493
type: string
25012494
example: |
25022495
[core]
25032496
dags_folder = /home/user/my-dags-folder
2504-
[smtp]
2505-
smtp_host = localhost
2506-
smtp_mail_from = [email protected]
25072497
25082498
"401":
25092499
$ref: "#/components/responses/Unauthenticated"

airflow/api_fastapi/core_api/openapi/v1-generated.yaml

Lines changed: 0 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -2672,13 +2672,6 @@ paths:
26722672
26732673
base_log_folder = /opt/airflow/logs
26742674
2675-
2676-
[smtp]
2677-
2678-
smtp_host = localhost
2679-
2680-
smtp_mail_from = [email protected]
2681-
26822675
'
26832676
'401':
26842677
content:

airflow/api_fastapi/core_api/routes/public/config.py

Lines changed: 0 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -56,10 +56,6 @@
5656
[core]
5757
dags_folder = /opt/airflow/dags
5858
base_log_folder = /opt/airflow/logs
59-
60-
[smtp]
61-
smtp_host = localhost
62-
smtp_mail_from = [email protected]
6359
"""
6460
),
6561
},

airflow/config_templates/unit_tests.cfg

Lines changed: 0 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -66,10 +66,6 @@ allowed_deserialization_classes = airflow.* tests.*
6666
# celery tests rely on it being set
6767
celery_logging_level = INFO
6868

69-
[smtp]
70-
# Used as default values for SMTP unit tests
71-
smtp_mail_from = [email protected]
72-
7369
[api]
7470
auth_backends = airflow.providers.fab.auth_manager.api.auth.backend.session
7571

airflow/configuration.py

Lines changed: 0 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -353,13 +353,6 @@ def inversed_deprecated_sections(self):
353353
"navbar_color": (re2.compile(r"(?i)\A#007A87\z"), "#fff", "2.1"),
354354
"dag_default_view": (re2.compile(r"^tree$"), "grid", "3.0"),
355355
},
356-
"email": {
357-
"email_backend": (
358-
re2.compile(r"^airflow\.contrib\.utils\.sendgrid\.send_email$"),
359-
r"airflow.providers.sendgrid.utils.emailer.send_email",
360-
"2.1",
361-
),
362-
},
363356
"logging": {
364357
"log_filename_template": (
365358
re2.compile(re2.escape("{{ ti.dag_id }}/{{ ti.task_id }}/{{ ts }}/{{ try_number }}.log")),

airflow/example_dags/example_dag_decorator.py

Lines changed: 9 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -24,7 +24,7 @@
2424

2525
from airflow.decorators import dag, task
2626
from airflow.models.baseoperator import BaseOperator
27-
from airflow.operators.email import EmailOperator
27+
from airflow.providers.smtp.operators.smtp import EmailOperator
2828

2929
if TYPE_CHECKING:
3030
from airflow.sdk.definitions.context import Context
@@ -48,11 +48,12 @@ def execute(self, context: Context):
4848
catchup=False,
4949
tags=["example"],
5050
)
51-
def example_dag_decorator(email: str = "[email protected]"):
51+
def example_dag_decorator(to_email: str = "example1@example.com", from_email: str = "example2@example.com"):
5252
"""
5353
DAG to send server IP to email.
5454
55-
:param email: Email to send IP to. Defaults to [email protected].
55+
:param to_email: Email to send IP to. Defaults to [email protected].
56+
:param from_email: Email to send IP from. Defaults to [email protected]
5657
"""
5758
get_ip = GetRequestOperator(task_id="get_ip", url="http://httpbin.org/get")
5859

@@ -67,7 +68,11 @@ def prepare_email(raw_json: dict[str, Any]) -> dict[str, str]:
6768
email_info = prepare_email(get_ip.output)
6869

6970
EmailOperator(
70-
task_id="send_email", to=email, subject=email_info["subject"], html_content=email_info["body"]
71+
task_id="send_email",
72+
to=to_email,
73+
from_email=from_email,
74+
subject=email_info["subject"],
75+
html_content=email_info["body"],
7176
)
7277

7378

airflow/example_dags/tutorial.py

Lines changed: 0 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -45,9 +45,6 @@
4545
# You can override them on a per-task basis during operator initialization
4646
default_args={
4747
"depends_on_past": False,
48-
"email": ["[email protected]"],
49-
"email_on_failure": False,
50-
"email_on_retry": False,
5148
"retries": 1,
5249
"retry_delay": timedelta(minutes=5),
5350
# 'queue': 'bash_queue',

airflow/models/baseoperator.py

Lines changed: 0 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -212,13 +212,6 @@ class derived from this one results in the creation of a task object,
212212
:param task_id: a unique, meaningful id for the task
213213
:param owner: the owner of the task. Using a meaningful description
214214
(e.g. user/person/team/role name) to clarify ownership is recommended.
215-
:param email: the 'to' email address(es) used in email alerts. This can be a
216-
single email or multiple ones. Multiple addresses can be specified as a
217-
comma or semicolon separated string or by passing a list of strings.
218-
:param email_on_retry: Indicates whether email alerts should be sent when a
219-
task is retried
220-
:param email_on_failure: Indicates whether email alerts should be sent when
221-
a task failed
222215
:param retries: the number of retries that should be performed before
223216
failing the task
224217
:param retry_delay: delay between retries, can be set as ``timedelta`` or

airflow/models/taskinstance.py

Lines changed: 3 additions & 150 deletions
Original file line numberDiff line numberDiff line change
@@ -23,7 +23,6 @@
2323
import itertools
2424
import logging
2525
import math
26-
import operator
2726
import os
2827
import signal
2928
import traceback
@@ -106,7 +105,6 @@
106105
from airflow.models.xcom import LazyXComSelectSequence, XCom
107106
from airflow.plugins_manager import integrate_macros_plugins
108107
from airflow.sdk.api.datamodels._generated import AssetProfile
109-
from airflow.sdk.definitions._internal.templater import SandboxedEnvironment
110108
from airflow.sdk.definitions.asset import Asset, AssetAlias, AssetNameRef, AssetUniqueKey, AssetUriRef
111109
from airflow.sdk.definitions.taskgroup import MappedTaskGroup
112110
from airflow.sentry import Sentry
@@ -123,10 +121,8 @@
123121
OutletEventAccessors,
124122
VariableAccessor,
125123
context_get_outlet_events,
126-
context_merge,
127124
)
128-
from airflow.utils.email import send_email
129-
from airflow.utils.helpers import prune_dict, render_template_to_string
125+
from airflow.utils.helpers import prune_dict
130126
from airflow.utils.log.logging_mixin import LoggingMixin
131127
from airflow.utils.net import get_hostname
132128
from airflow.utils.operator_helpers import ExecutionCallableRunner, context_to_airflow_vars
@@ -1117,15 +1113,6 @@ def _handle_failure(
11171113
)
11181114

11191115
_log_state(task_instance=task_instance, lead_msg="Immediate failure requested. " if force_fail else "")
1120-
if (
1121-
failure_context["task"]
1122-
and failure_context["email_for_state"](failure_context["task"])
1123-
and failure_context["task"].email
1124-
):
1125-
try:
1126-
task_instance.email_alert(error, failure_context["task"])
1127-
except Exception:
1128-
log.exception("Failed to send email to: %s", failure_context["task"].email)
11291116

11301117
if failure_context["callbacks"] and failure_context["context"]:
11311118
_run_finished_callback(
@@ -1317,116 +1304,6 @@ def _get_previous_start_date(
13171304
return pendulum.instance(prev_ti.start_date) if prev_ti and prev_ti.start_date else None
13181305

13191306

1320-
def _email_alert(*, task_instance: TaskInstance, exception, task: BaseOperator) -> None:
1321-
"""
1322-
Send alert email with exception information.
1323-
1324-
:param task_instance: the task instance
1325-
:param exception: the exception
1326-
:param task: task related to the exception
1327-
1328-
:meta private:
1329-
"""
1330-
subject, html_content, html_content_err = task_instance.get_email_subject_content(exception, task=task)
1331-
if TYPE_CHECKING:
1332-
assert task.email
1333-
try:
1334-
send_email(task.email, subject, html_content)
1335-
except Exception:
1336-
send_email(task.email, subject, html_content_err)
1337-
1338-
1339-
def _get_email_subject_content(
1340-
*,
1341-
task_instance: TaskInstance,
1342-
exception: BaseException,
1343-
task: BaseOperator | None = None,
1344-
) -> tuple[str, str, str]:
1345-
"""
1346-
Get the email subject content for exceptions.
1347-
1348-
:param task_instance: the task instance
1349-
:param exception: the exception sent in the email
1350-
:param task:
1351-
1352-
:meta private:
1353-
"""
1354-
# For a ti from DB (without ti.task), return the default value
1355-
if task is None:
1356-
task = getattr(task_instance, "task")
1357-
use_default = task is None
1358-
exception_html = str(exception).replace("\n", "<br>")
1359-
1360-
default_subject = "Airflow alert: {{ti}}"
1361-
# For reporting purposes, we report based on 1-indexed,
1362-
# not 0-indexed lists (i.e. Try 1 instead of
1363-
# Try 0 for the first attempt).
1364-
default_html_content = (
1365-
"Try {{try_number}} out of {{max_tries + 1}}<br>"
1366-
"Exception:<br>{{exception_html}}<br>"
1367-
'Log: <a href="{{ti.log_url}}">Link</a><br>'
1368-
"Host: {{ti.hostname}}<br>"
1369-
'Mark success: <a href="{{ti.mark_success_url}}">Link</a><br>'
1370-
)
1371-
1372-
default_html_content_err = (
1373-
"Try {{try_number}} out of {{max_tries + 1}}<br>"
1374-
"Exception:<br>Failed attempt to attach error logs<br>"
1375-
'Log: <a href="{{ti.log_url}}">Link</a><br>'
1376-
"Host: {{ti.hostname}}<br>"
1377-
'Mark success: <a href="{{ti.mark_success_url}}">Link</a><br>'
1378-
)
1379-
1380-
additional_context: dict[str, Any] = {
1381-
"exception": exception,
1382-
"exception_html": exception_html,
1383-
"try_number": task_instance.try_number,
1384-
"max_tries": task_instance.max_tries,
1385-
}
1386-
1387-
if use_default:
1388-
default_context = {"ti": task_instance, **additional_context}
1389-
jinja_env = jinja2.Environment(
1390-
loader=jinja2.FileSystemLoader(os.path.dirname(__file__)), autoescape=True
1391-
)
1392-
subject = jinja_env.from_string(default_subject).render(**default_context)
1393-
html_content = jinja_env.from_string(default_html_content).render(**default_context)
1394-
html_content_err = jinja_env.from_string(default_html_content_err).render(**default_context)
1395-
1396-
else:
1397-
if TYPE_CHECKING:
1398-
assert task_instance.task
1399-
1400-
# Use the DAG's get_template_env() to set force_sandboxed. Don't add
1401-
# the flag to the function on task object -- that function can be
1402-
# overridden, and adding a flag breaks backward compatibility.
1403-
dag = task_instance.task.get_dag()
1404-
if dag:
1405-
jinja_env = dag.get_template_env(force_sandboxed=True)
1406-
else:
1407-
jinja_env = SandboxedEnvironment(cache_size=0)
1408-
jinja_context = task_instance.get_template_context()
1409-
context_merge(jinja_context, additional_context)
1410-
1411-
def render(key: str, content: str) -> str:
1412-
if conf.has_option("email", key):
1413-
path = conf.get_mandatory_value("email", key)
1414-
try:
1415-
with open(path) as f:
1416-
content = f.read()
1417-
except FileNotFoundError:
1418-
log.warning("Could not find email template file '%s'. Using defaults...", path)
1419-
except OSError:
1420-
log.exception("Error while using email template %s. Using defaults...", path)
1421-
return render_template_to_string(jinja_env.from_string(content), jinja_context)
1422-
1423-
subject = render("subject_template", default_subject)
1424-
html_content = render("html_content_template", default_html_content)
1425-
html_content_err = render("html_content_template", default_html_content_err)
1426-
1427-
return subject, html_content, html_content_err
1428-
1429-
14301307
def _run_finished_callback(
14311308
*,
14321309
callbacks: None | TaskStateChangeCallback | list[TaskStateChangeCallback],
@@ -3131,8 +3008,7 @@ def fetch_handle_failure_context(
31313008
if context is not None:
31323009
context["exception"] = error
31333010

3134-
# Set state correctly and figure out how to log it and decide whether
3135-
# to email
3011+
# Set state correctly and figure out how to log it
31363012

31373013
# Note, callback invocation needs to be handled by caller of
31383014
# _run_raw_task to avoid race conditions which could lead to duplicate
@@ -3150,11 +3026,10 @@ def fetch_handle_failure_context(
31503026
assert isinstance(ti.task, BaseOperator)
31513027
task = ti.task.unmap((context, session))
31523028
except Exception:
3153-
cls.logger().error("Unable to unmap task to determine if we need to send an alert email")
3029+
cls.logger().error("Unable to unmap task to determine what callback to use")
31543030

31553031
if force_fail or not ti.is_eligible_to_retry():
31563032
ti.state = TaskInstanceState.FAILED
3157-
email_for_state = operator.attrgetter("email_on_failure")
31583033
callbacks = task.on_failure_callback if task else None
31593034

31603035
if task and fail_fast:
@@ -3169,7 +3044,6 @@ def fetch_handle_failure_context(
31693044
TaskInstanceHistory.record_ti(ti, session=session)
31703045

31713046
ti.state = State.UP_FOR_RETRY
3172-
email_for_state = operator.attrgetter("email_on_retry")
31733047
callbacks = task.on_retry_callback if task else None
31743048

31753049
get_listener_manager().hook.on_task_instance_failed(
@@ -3178,7 +3052,6 @@ def fetch_handle_failure_context(
31783052

31793053
return {
31803054
"ti": ti,
3181-
"email_for_state": email_for_state,
31823055
"task": task,
31833056
"callbacks": callbacks,
31843057
"context": context,
@@ -3326,26 +3199,6 @@ def render_templates(
33263199

33273200
return original_task
33283201

3329-
def get_email_subject_content(
3330-
self, exception: BaseException, task: BaseOperator | None = None
3331-
) -> tuple[str, str, str]:
3332-
"""
3333-
Get the email subject content for exceptions.
3334-
3335-
:param exception: the exception sent in the email
3336-
:param task:
3337-
"""
3338-
return _get_email_subject_content(task_instance=self, exception=exception, task=task)
3339-
3340-
def email_alert(self, exception, task: BaseOperator) -> None:
3341-
"""
3342-
Send alert email with exception information.
3343-
3344-
:param exception: the exception
3345-
:param task: task related to the exception
3346-
"""
3347-
_email_alert(task_instance=self, exception=exception, task=task)
3348-
33493202
def set_duration(self) -> None:
33503203
"""Set task instance duration."""
33513204
_set_duration(task_instance=self)

airflow/models/trigger.py

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -261,7 +261,7 @@ def submit_failure(cls, trigger_id, exc=None, session: Session = NEW_SESSION) ->
261261
the runtime code understands as immediate-fail, and pack the error into
262262
next_kwargs.
263263
264-
TODO: Once we have shifted callback (and email) handling to run on
264+
TODO: Once we have shifted callback handling to run on
265265
workers as first-class concepts, we can run the failure code here
266266
in-process, but we can't do that right now.
267267
"""

0 commit comments

Comments
 (0)