## 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.importdatetimeimportfunctoolsimporthashlibimporttimeimportwarningsfromdatetimeimporttimedeltafromtypingimportAny,Callable,Iterable,Optional,Unionfromairflowimportsettingsfromairflow.configurationimportconffromairflow.exceptionsimport(AirflowException,AirflowRescheduleException,AirflowSensorTimeout,AirflowSkipException,)fromairflow.models.baseoperatorimportBaseOperatorfromairflow.models.sensorinstanceimportSensorInstancefromairflow.models.skipmixinimportSkipMixinfromairflow.models.taskrescheduleimportTaskReschedulefromairflow.ti_deps.deps.ready_to_rescheduleimportReadyToRescheduleDepfromairflow.utilsimporttimezonefromairflow.utils.contextimportContext# We need to keep the import here because GCSToLocalFilesystemOperator released in# Google Provider before 3.0.0 imported apply_defaults from here.# See https://github.com/apache/airflow/issues/16035fromairflow.utils.decoratorsimportapply_defaults# noqa: F401fromairflow.utils.docsimportget_docs_url# As documented in https://dev.mysql.com/doc/refman/5.7/en/datetime.html._MYSQL_TIMESTAMP_MAX=datetime.datetime(2038,1,19,3,14,7,tzinfo=timezone.utc)@functools.lru_cache(maxsize=None)def_is_metadatabase_mysql()->bool:ifsettings.engineisNone:raiseAirflowException("Must initialize ORM first")returnsettings.engine.url.get_backend_name()=="mysql"
[docs]classPokeReturnValue:""" Sensors can optionally return an instance of the PokeReturnValue class in the poke method. If an XCom value is supplied when the sensor is done, then the XCom value will be pushed through the operator return value. :param is_done: Set to true to indicate the sensor can stop poking. :param xcom_value: An optional XCOM value to be returned by the operator. """def__init__(self,is_done:bool,xcom_value:Optional[Any]=None)->None:self.xcom_value=xcom_valueself.is_done=is_done
[docs]classBaseSensorOperator(BaseOperator,SkipMixin):""" Sensor operators are derived from this class and inherit these attributes. Sensor operators keep executing at a time interval and succeed when a criteria is met and fail if and when they time out. :param soft_fail: Set to true to mark the task as SKIPPED on failure :param poke_interval: Time in seconds that the job should wait in between each tries :param timeout: Time, in seconds before the task times out and fails. :param mode: How the sensor operates. Options are: ``{ poke | reschedule }``, default is ``poke``. When set to ``poke`` the sensor is taking up a worker slot for its whole execution time and sleeps between pokes. Use this mode if the expected runtime of the sensor is short or if a short poke interval is required. Note that the sensor will hold onto a worker slot and a pool slot for the duration of the sensor's runtime in this mode. When set to ``reschedule`` the sensor task frees the worker slot when the criteria is not yet met and it's rescheduled at a later time. Use this mode if the time before the criteria is met is expected to be quite long. The poke interval should be more than one minute to prevent too much load on the scheduler. :param exponential_backoff: allow progressive longer waits between pokes by using exponential backoff algorithm """
# As the poke context in smart sensor defines the poking job signature only,# The execution_fields defines other execution details# for this tasks such as the customer defined timeout, the email and the alert# setup. Smart sensor serialize these attributes into a different DB column so# that smart sensor service is able to handle corresponding execution details# without breaking the sensor poking logic with dedup.
def__init__(self,*,poke_interval:float=60,timeout:float=conf.getfloat('sensors','default_timeout'),soft_fail:bool=False,mode:str='poke',exponential_backoff:bool=False,**kwargs,)->None:super().__init__(**kwargs)self.poke_interval=poke_intervalself.soft_fail=soft_failself.timeout=timeoutself.mode=modeself.exponential_backoff=exponential_backoffself._validate_input_values()self.sensor_service_enabled=conf.getboolean('smart_sensor','use_smart_sensor')self.sensors_support_sensor_service=set(map(lambdal:l.strip(),conf.get_mandatory_value('smart_sensor','sensors_enabled').split(',')))def_validate_input_values(self)->None:ifnotisinstance(self.poke_interval,(int,float))orself.poke_interval<0:raiseAirflowException("The poke_interval must be a non-negative number")ifnotisinstance(self.timeout,(int,float))orself.timeout<0:raiseAirflowException("The timeout must be a non-negative number")ifself.modenotinself.valid_modes:raiseAirflowException(f"The mode must be one of {self.valid_modes},'{self.dag.dag_idifself.has_dag()else''} "f".{self.task_id}'; received '{self.mode}'.")# Quick check for poke_interval isn't immediately over MySQL's TIMESTAMP limit.# This check is only rudimentary to catch trivial user errors, e.g. mistakenly# set the value to milliseconds instead of seconds. There's another check when# we actually try to reschedule to ensure database coherence.ifself.rescheduleand_is_metadatabase_mysql():iftimezone.utcnow()+datetime.timedelta(seconds=self.poke_interval)>_MYSQL_TIMESTAMP_MAX:raiseAirflowException(f"Cannot set poke_interval to {self.poke_interval} seconds in reschedule "f"mode since it will take reschedule time over MySQL's TIMESTAMP limit.")
[docs]defpoke(self,context:Context)->Union[bool,PokeReturnValue]:""" Function that the sensors defined while deriving this class should override. """raiseAirflowException('Override me.')
[docs]defregister_in_sensor_service(self,ti,context):""" Register ti in smart sensor service :param ti: Task instance object. :param context: TaskInstance template context from the ti. :return: boolean """docs_url=get_docs_url('concepts/smart-sensors.html#migrating-to-deferrable-operators')warnings.warn('Your sensor is using Smart Sensors, which are deprecated.'f' Please use Deferrable Operators instead. See {docs_url} for more info.',DeprecationWarning,)poke_context=self.get_poke_context(context)execution_context=self.get_execution_context(context)returnSensorInstance.register(ti,poke_context,execution_context)
[docs]defget_poke_context(self,context):""" Return a dictionary with all attributes in poke_context_fields. The poke_context with operator class can be used to identify a unique sensor job. :param context: TaskInstance template context. :return: A dictionary with key in poke_context_fields. """ifnotcontext:self.log.info("Function get_poke_context doesn't have a context input.")poke_context_fields=getattr(self.__class__,"poke_context_fields",None)result={key:getattr(self,key,None)forkeyinpoke_context_fields}returnresult
[docs]defget_execution_context(self,context):""" Return a dictionary with all attributes in execution_fields. The execution_context include execution requirement for each sensor task such as timeout setup, email_alert setup. :param context: TaskInstance template context. :return: A dictionary with key in execution_fields. """ifnotcontext:self.log.info("Function get_execution_context doesn't have a context input.")execution_fields=self.__class__.execution_fieldsresult={key:getattr(self,key,None)forkeyinexecution_fields}ifresult['execution_timeout']andisinstance(result['execution_timeout'],datetime.timedelta):result['execution_timeout']=result['execution_timeout'].total_seconds()returnresult
[docs]defexecute(self,context:Context)->Any:started_at:Union[datetime.datetime,float]ifself.reschedule:# If reschedule, use the start date of the first try (first try can be either the very# first execution of the task, or the first execution after the task was cleared.)first_try_number=context['ti'].max_tries-self.retries+1task_reschedules=TaskReschedule.find_for_task_instance(context['ti'],try_number=first_try_number)ifnottask_reschedules:start_date=timezone.utcnow()else:start_date=task_reschedules[0].start_datestarted_at=start_datedefrun_duration()->float:# If we are in reschedule mode, then we have to compute diff# based on the time in a DB, so can't use time.monotonicreturn(timezone.utcnow()-start_date).total_seconds()else:started_at=start_monotonic=time.monotonic()defrun_duration()->float:returntime.monotonic()-start_monotonictry_number=1log_dag_id=self.dag.dag_idifself.has_dag()else""xcom_value=NonewhileTrue:poke_return=self.poke(context)ifpoke_return:ifisinstance(poke_return,PokeReturnValue):xcom_value=poke_return.xcom_valuebreakifrun_duration()>self.timeout:# If sensor is in soft fail mode but times out raise AirflowSkipException.ifself.soft_fail:raiseAirflowSkipException(f"Snap. Time is OUT. DAG id: {log_dag_id}")else:raiseAirflowSensorTimeout(f"Snap. Time is OUT. DAG id: {log_dag_id}")ifself.reschedule:next_poke_interval=self._get_next_poke_interval(started_at,run_duration,try_number)reschedule_date=timezone.utcnow()+timedelta(seconds=next_poke_interval)if_is_metadatabase_mysql()andreschedule_date>_MYSQL_TIMESTAMP_MAX:raiseAirflowSensorTimeout(f"Cannot reschedule DAG {log_dag_id} to {reschedule_date.isoformat()} "f"since it is over MySQL's TIMESTAMP storage limit.")raiseAirflowRescheduleException(reschedule_date)else:time.sleep(self._get_next_poke_interval(started_at,run_duration,try_number))try_number+=1self.log.info("Success criteria met. Exiting.")returnxcom_value
def_get_next_poke_interval(self,started_at:Union[datetime.datetime,float],run_duration:Callable[[],float],try_number:int,)->float:"""Using the similar logic which is used for exponential backoff retry delay for operators."""ifnotself.exponential_backoff:returnself.poke_intervalmin_backoff=int(self.poke_interval*(2**(try_number-2)))run_hash=int(hashlib.sha1(f"{self.dag_id}#{self.task_id}#{started_at}#{try_number}".encode()).hexdigest(),16,)modded_hash=min_backoff+run_hash%min_backoffdelay_backoff_in_seconds=min(modded_hash,timedelta.max.total_seconds()-1)new_interval=min(self.timeout-int(run_duration()),delay_backoff_in_seconds)self.log.info("new %s interval is %s",self.mode,new_interval)returnnew_interval
[docs]defprepare_for_execution(self)->BaseOperator:task=super().prepare_for_execution()# Sensors in `poke` mode can block execution of DAGs when running# with single process executor, thus we change the mode to`reschedule`# to allow parallel task being scheduled and executedifconf.get('core','executor')=="DebugExecutor":self.log.warning("DebugExecutor changes sensor mode to 'reschedule'.")task.mode='reschedule'returntask
[docs]defpoke_mode_only(cls):""" Class Decorator for child classes of BaseSensorOperator to indicate that instances of this class are only safe to use poke mode. Will decorate all methods in the class to assert they did not change the mode from 'poke'. :param cls: BaseSensor class to enforce methods only use 'poke' mode. """defdecorate(cls_type):defmode_getter(_):return'poke'defmode_setter(_,value):ifvalue!='poke':raiseValueError("cannot set mode to 'poke'.")ifnotissubclass(cls_type,BaseSensorOperator):raiseValueError(f"poke_mode_only decorator should only be "f"applied to subclasses of BaseSensorOperator,"f" got:{cls_type}.")cls_type.mode=property(mode_getter,mode_setter)returncls_typereturndecorate(cls)