Skip to content

Commit

Permalink
[AIRFLOW-31] Use standard imports for hooks/operators
Browse files Browse the repository at this point in the history
  • Loading branch information
jlowin committed Jun 16, 2016
1 parent 06e70e2 commit 851adc5
Show file tree
Hide file tree
Showing 71 changed files with 1,419 additions and 435 deletions.
6 changes: 3 additions & 3 deletions airflow/__init__.py
Original file line number Diff line number Diff line change
Expand Up @@ -79,6 +79,6 @@ def __init__(self, namespace):
from airflow import macros
from airflow import contrib

operators.integrate_plugins()
hooks.integrate_plugins()
macros.integrate_plugins()
operators._integrate_plugins()
hooks._integrate_plugins()
macros._integrate_plugins()
4 changes: 2 additions & 2 deletions airflow/contrib/example_dags/example_twitter_dag.py
Original file line number Diff line number Diff line change
Expand Up @@ -23,7 +23,8 @@
# --------------------------------------------------------------------------------

from airflow import DAG
from airflow.operators import BashOperator, HiveOperator, PythonOperator
from airflow.operators import BashOperator, PythonOperator
from airflow.operators.hive_operator import HiveOperator
from datetime import datetime, date, timedelta

# --------------------------------------------------------------------------------
Expand Down Expand Up @@ -180,4 +181,3 @@ def transfertodb():

load_to_hive.set_upstream(load_to_hdfs)
load_to_hive.set_downstream(hive_to_mysql)

33 changes: 32 additions & 1 deletion airflow/contrib/hooks/__init__.py
Original file line number Diff line number Diff line change
Expand Up @@ -11,7 +11,28 @@
# 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.



# Contrib hooks are not imported by default. They should be accessed
# directly: from airflow.contrib.hooks.hook_module import Hook







# ------------------------------------------------------------------------
#
# #TODO #FIXME Airflow 2.0
#
# Old import machinary below.
#
# This is deprecated but should be kept until Airflow 2.0
# for compatibility.
#
# ------------------------------------------------------------------------

# Imports the hooks dynamically while keeping the package API clean,
# abstracting the underlying modules
Expand All @@ -31,4 +52,14 @@
'fs_hook': ['FSHook']
}

_import_module_attrs(globals(), _hooks)
import os as _os
if not _os.environ.get('AIRFLOW_USE_NEW_IMPORTS', False):
from zope.deprecation import deprecated as _deprecated
_imported = _import_module_attrs(globals(), _hooks)
for _i in _imported:
_deprecated(
_i,
"Importing {i} directly from 'contrib.hooks' has been "
"deprecated. Please import from "
"'contrib.hooks.[hook_module]' instead. Support for direct imports "
"will be dropped entirely in Airflow 2.0.".format(i=_i))
42 changes: 41 additions & 1 deletion airflow/contrib/operators/__init__.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,33 @@
# -*- coding: utf-8 -*-
#
# Licensed 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.


# Contrib operators are not imported by default. They should be accessed
# directly: from airflow.contrib.operators.operator_module import Operator


# ------------------------------------------------------------------------
#
# #TODO #FIXME Airflow 2.0
#
# Old import machinary below.
#
# This is deprecated but should be kept until Airflow 2.0
# for compatibility.
#
# ------------------------------------------------------------------------

# Imports the operators dynamically while keeping the package API clean,
# abstracting the underlying modules
from airflow.utils.helpers import import_module_attrs as _import_module_attrs
Expand All @@ -10,4 +40,14 @@
'fs': ['FileSensor']
}

_import_module_attrs(globals(), _operators)
import os as _os
if not _os.environ.get('AIRFLOW_USE_NEW_IMPORTS', False):
from zope.deprecation import deprecated as _deprecated
_imported = _import_module_attrs(globals(), _operators)
for _i in _imported:
_deprecated(
_i,
"Importing {i} directly from 'contrib.operators' has been "
"deprecated. Please import from "
"'contrib.operators.[operator_module]' instead. Support for direct "
"imports will be dropped entirely in Airflow 2.0.".format(i=_i))
3 changes: 1 addition & 2 deletions airflow/contrib/operators/fs_operator.py
Original file line number Diff line number Diff line change
Expand Up @@ -17,7 +17,7 @@
import logging

from airflow.operators.sensors import BaseSensorOperator
from airflow.contrib.hooks import FSHook
from airflow.contrib.hooks.fs_hook import FSHook
from airflow.utils.decorators import apply_defaults

class FileSensor(BaseSensorOperator):
Expand Down Expand Up @@ -54,4 +54,3 @@ def poke(self, context):
except:
return False
return True

2 changes: 1 addition & 1 deletion airflow/contrib/operators/mysql_to_gcs.py
Original file line number Diff line number Diff line change
Expand Up @@ -3,7 +3,7 @@
import time

from airflow.contrib.hooks.gcs_hook import GoogleCloudStorageHook
from airflow.hooks import MySqlHook
from airflow.hooks.mysql_hook import MySqlHook
from airflow.models import BaseOperator
from airflow.utils.decorators import apply_defaults
from collections import OrderedDict
Expand Down
2 changes: 1 addition & 1 deletion airflow/contrib/operators/vertica_to_hive.py
Original file line number Diff line number Diff line change
Expand Up @@ -4,7 +4,7 @@
import logging
from tempfile import NamedTemporaryFile

from airflow.hooks import HiveCliHook
from airflow.hooks.hive_hooks import HiveCliHook
from airflow.contrib.hooks import VerticaHook
from airflow.models import BaseOperator
from airflow.utils.decorators import apply_defaults
Expand Down
4 changes: 3 additions & 1 deletion airflow/contrib/plugins/metastore_browser/main.py
Original file line number Diff line number Diff line change
Expand Up @@ -5,7 +5,9 @@
from flask_admin import BaseView, expose
import pandas as pd

from airflow.hooks import HiveMetastoreHook, MySqlHook, PrestoHook, HiveCliHook
from airflow.hooks.hive_hooks import HiveMetastoreHook, HiveCliHook
from airflow.hooks.mysql_hook import MySqlHook
from airflow.hooks.presto_hook import PrestoHook
from airflow.plugins_manager import AirflowPlugin
from airflow.www import utils as wwwutils

Expand Down
3 changes: 2 additions & 1 deletion airflow/example_dags/example_http_operator.py
Original file line number Diff line number Diff line change
Expand Up @@ -2,7 +2,8 @@
### Example HTTP operator and sensor
"""
from airflow import DAG
from airflow.operators import SimpleHttpOperator, HttpSensor
from airflow.operators import SimpleHttpOperator
from airflow.operators.sensors import HttpSensor
from datetime import datetime, timedelta
import json

Expand Down
73 changes: 67 additions & 6 deletions airflow/hooks/__init__.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,36 @@
# -*- coding: utf-8 -*-
#
# Licensed 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.


# Only import Core Airflow Operators that don't have extra requirements.
# All other operators must be imported directly.
from .base_hook import BaseHook
from .dbapi_hook import DbApiHook
from .http_hook import HttpHook
from .sqlite_hook import SqliteHook

# ------------------------------------------------------------------------
#
# #TODO #FIXME Airflow 2.0
#
# Old import machinary below.
#
# This is deprecated but should be kept until Airflow 2.0
# for compatibility.
#
# ------------------------------------------------------------------------

# Imports the hooks dynamically while keeping the package API clean,
# abstracting the underlying modules

Expand All @@ -17,21 +50,49 @@
'postgres_hook': ['PostgresHook'],
'presto_hook': ['PrestoHook'],
'samba_hook': ['SambaHook'],
'sqlite_hook': ['SqliteHook'],
# 'sqlite_hook': ['SqliteHook'],
'S3_hook': ['S3Hook'],
'http_hook': ['HttpHook'],
# 'http_hook': ['HttpHook'],
'druid_hook': ['DruidHook'],
'jdbc_hook': ['JdbcHook'],
'dbapi_hook': ['DbApiHook'],
'mssql_hook': ['MsSqlHook'],
'oracle_hook': ['OracleHook'],
}

_import_module_attrs(globals(), _hooks)

import os as _os
if not _os.environ.get('AIRFLOW_USE_NEW_IMPORTS', False):
from zope.deprecation import deprecated as _deprecated
_imported = _import_module_attrs(globals(), _hooks)
for _i in _imported:
_deprecated(
_i,
"Importing {i} directly from 'airflow.hooks' has been "
"deprecated. Please import from "
"'airflow.hooks.[hook_module]' instead. Support for direct imports "
"will be dropped entirely in Airflow 2.0.".format(i=_i))

def integrate_plugins():

def _integrate_plugins():
"""Integrate plugins to the context"""
import sys
from airflow.plugins_manager import hooks as _hooks
for _h in _hooks:
globals()[_h.__name__] = _h
for _hook_module in _hooks:
sys.modules[_hook_module.__name__] = _hook_module
globals()[_hook_module._name] = _hook_module


##########################################################
# TODO FIXME Remove in Airflow 2.0

if not _os.environ.get('AIRFLOW_USE_NEW_IMPORTS', False):
from zope.deprecation import deprecated as _deprecated
for _hook in _hook_module._objects:
globals()[_hook.__name__] = _deprecated(
_hook,
"Importing plugin hook '{i}' directly from "
"'airflow.hooks' has been deprecated. Please "
"import from 'airflow.hooks.[plugin_module]' "
"instead. Support for direct imports will be dropped "
"entirely in Airflow 2.0.".format(i=_hook))
14 changes: 14 additions & 0 deletions airflow/hooks/base_hook.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,17 @@
# -*- coding: utf-8 -*-
#
# Licensed 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 __future__ import absolute_import
from __future__ import division
from __future__ import print_function
Expand Down
14 changes: 14 additions & 0 deletions airflow/hooks/dbapi_hook.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,17 @@
# -*- coding: utf-8 -*-
#
# Licensed 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 builtins import str
from past.builtins import basestring
Expand Down
14 changes: 14 additions & 0 deletions airflow/hooks/druid_hook.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,17 @@
# -*- coding: utf-8 -*-
#
# Licensed 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 __future__ import print_function
import logging
import json
Expand Down
14 changes: 14 additions & 0 deletions airflow/hooks/hdfs_hook.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,17 @@
# -*- coding: utf-8 -*-
#
# Licensed 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.hooks.base_hook import BaseHook
from airflow import configuration

Expand Down
2 changes: 1 addition & 1 deletion airflow/hooks/http_hook.py
Original file line number Diff line number Diff line change
Expand Up @@ -11,7 +11,7 @@
# 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 builtins import str
import logging

Expand Down
14 changes: 14 additions & 0 deletions airflow/hooks/jdbc_hook.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,17 @@
# -*- coding: utf-8 -*-
#
# Licensed 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 builtins import str
__author__ = 'janomar'

Expand Down
14 changes: 14 additions & 0 deletions airflow/hooks/mssql_hook.py
Original file line number Diff line number Diff line change
@@ -1,3 +1,17 @@
# -*- coding: utf-8 -*-
#
# Licensed 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 pymssql

from airflow.hooks.dbapi_hook import DbApiHook
Expand Down
Loading

0 comments on commit 851adc5

Please sign in to comment.