[ 
https://issues.apache.org/jira/browse/AIRFLOW-3250?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16698335#comment-16698335
 ] 

ASF GitHub Bot commented on AIRFLOW-3250:
-----------------------------------------

Fokko closed pull request #4090: [AIRFLOW-3250] Fix for Redis Hook for not 
authorised connection calls
URL: https://github.com/apache/incubator-airflow/pull/4090
 
 
   

This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:

As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):

diff --git a/airflow/contrib/hooks/redis_hook.py 
b/airflow/contrib/hooks/redis_hook.py
index 650cc9308b..a34e880796 100644
--- a/airflow/contrib/hooks/redis_hook.py
+++ b/airflow/contrib/hooks/redis_hook.py
@@ -21,15 +21,13 @@
 RedisHook module
 """
 from redis import StrictRedis
-
-from airflow.exceptions import AirflowException
 from airflow.hooks.base_hook import BaseHook
 from airflow.utils.log.logging_mixin import LoggingMixin
 
 
 class RedisHook(BaseHook, LoggingMixin):
     """
-    Hook to interact with Redis database
+    Wrapper for connection to interact with Redis in-memory data structure 
store
     """
     def __init__(self, redis_conn_id='redis_default'):
         """
@@ -39,55 +37,31 @@ def __init__(self, redis_conn_id='redis_default'):
                             we need to connect to Redis.
         """
         self.redis_conn_id = redis_conn_id
-        self.client = None
-        conn = self.get_connection(self.redis_conn_id)
-        self.host = conn.host
-        self.port = int(conn.port)
-        self.password = conn.password
-        self.db = int(conn.extra_dejson.get('db', 0))
-
-        self.log.debug(
-            '''Connection "{conn}":
-            \thost: {host}
-            \tport: {port}
-            \textra: {extra}
-            '''.format(
-                conn=self.redis_conn_id,
-                host=self.host,
-                port=self.port,
-                extra=conn.extra_dejson
-            )
-        )
+        self.redis = None
+        self.host = None
+        self.port = None
+        self.password = None
+        self.db = None
 
     def get_conn(self):
         """
         Returns a Redis connection.
         """
-        if not self.client:
+        conn = self.get_connection(self.redis_conn_id)
+        self.host = conn.host
+        self.port = conn.port
+        self.password = None if str(conn.password).lower() in ['none', 
'false', ''] else conn.password
+        self.db = conn.extra_dejson.get('db', None)
+
+        if not self.redis:
             self.log.debug(
-                'generating Redis client for conn_id "%s" on %s:%s:%s',
+                'Initializing redis object for conn_id "%s" on %s:%s:%s',
                 self.redis_conn_id, self.host, self.port, self.db
             )
-            try:
-                self.client = StrictRedis(
-                    host=self.host,
-                    port=self.port,
-                    password=self.password,
-                    db=self.db)
-            except Exception as general_error:
-                raise AirflowException(
-                    'Failed to create Redis client, error: {error}'.format(
-                        error=str(general_error)
-                    )
-                )
-
-        return self.client
-
-    def key_exists(self, key):
-        """
-        Checks if a key exists in Redis database
+            self.redis = StrictRedis(
+                host=self.host,
+                port=self.port,
+                password=self.password,
+                db=self.db)
 
-        :param key: The key to check the existence.
-        :type key: str
-        """
-        return self.get_conn().exists(key)
+        return self.redis
diff --git a/airflow/contrib/sensors/redis_key_sensor.py 
b/airflow/contrib/sensors/redis_key_sensor.py
index a2d190baae..4c0ac68840 100644
--- a/airflow/contrib/sensors/redis_key_sensor.py
+++ b/airflow/contrib/sensors/redis_key_sensor.py
@@ -23,25 +23,17 @@
 
 class RedisKeySensor(BaseSensorOperator):
     """
-    Checks for the existence of a key in a Redis database
+    Checks for the existence of a key in a Redis
     """
     template_fields = ('key',)
     ui_color = '#f0eee4'
 
     @apply_defaults
     def __init__(self, key, redis_conn_id, *args, **kwargs):
-        """
-        Create a new RedisKeySensor
-
-        :param key: The key to be monitored
-        :type key: str
-        :param redis_conn_id: The connection ID to use when connecting to 
Redis DB.
-        :type redis_conn_id: str
-        """
         super(RedisKeySensor, self).__init__(*args, **kwargs)
         self.redis_conn_id = redis_conn_id
         self.key = key
 
     def poke(self, context):
-        self.log.info('Sensor check existence of key: %s', self.key)
-        return RedisHook(self.redis_conn_id).key_exists(self.key)
+        self.log.info('Sensor checks for existence of key: %s', self.key)
+        return RedisHook(self.redis_conn_id).get_conn().exists(self.key)
diff --git a/airflow/utils/db.py b/airflow/utils/db.py
index ffc41115a1..b6a807c86c 100644
--- a/airflow/utils/db.py
+++ b/airflow/utils/db.py
@@ -208,7 +208,7 @@ def initdb(rbac=False):
     merge_conn(
         models.Connection(
             conn_id='redis_default', conn_type='redis',
-            host='localhost', port=6379,
+            host='redis', port=6379,
             extra='{"db": 0}'))
     merge_conn(
         models.Connection(
diff --git a/scripts/ci/docker-compose.yml b/scripts/ci/docker-compose.yml
index 101ad95297..6c530ad335 100644
--- a/scripts/ci/docker-compose.yml
+++ b/scripts/ci/docker-compose.yml
@@ -45,6 +45,10 @@ services:
     image: rabbitmq:3.7
     restart: always
 
+  redis:
+    image: redis:5.0.1
+    restart: always
+
   openldap:
     image: osixia/openldap:1.2.0
     restart: always
@@ -87,6 +91,7 @@ services:
       - mongo
       - cassandra
       - rabbitmq
+      - redis
       - openldap
       - krb5-kdc-server
     volumes:
diff --git a/tests/contrib/hooks/test_redis_hook.py 
b/tests/contrib/hooks/test_redis_hook.py
index 12c30680e1..978b304d41 100644
--- a/tests/contrib/hooks/test_redis_hook.py
+++ b/tests/contrib/hooks/test_redis_hook.py
@@ -19,32 +19,43 @@
 
 
 import unittest
-from mock import patch
-
 from airflow import configuration
 from airflow.contrib.hooks.redis_hook import RedisHook
 
 
 class TestRedisHook(unittest.TestCase):
+
     def setUp(self):
         configuration.load_test_config()
 
     def test_get_conn(self):
         hook = RedisHook(redis_conn_id='redis_default')
-        self.assertEqual(hook.client, None)
-        self.assertEqual(
-            repr(hook.get_conn()),
-            (
-                'StrictRedis<ConnectionPool'
-                '<Connection<host=localhost,port=6379,db=0>>>'
-            )
-        )
-
-    @patch("airflow.contrib.hooks.redis_hook.RedisHook.get_conn")
-    def test_first_conn_instantiation(self, get_conn):
+        self.assertEqual(hook.redis, None)
+
+        self.assertEqual(hook.host, None, 'host initialised as None.')
+        self.assertEqual(hook.port, None, 'port initialised as None.')
+        self.assertEqual(hook.password, None, 'password initialised as None.')
+        self.assertEqual(hook.db, None, 'db initialised as None.')
+        self.assertIs(hook.get_conn(), hook.get_conn(), 'Connection 
initialized only if None.')
+
+    def test_get_conn_password_stays_none(self):
+        hook = RedisHook(redis_conn_id='redis_default')
+        hook.get_conn()
+        self.assertEqual(hook.password, None)
+
+    def test_real_ping(self):
         hook = RedisHook(redis_conn_id='redis_default')
-        hook.key_exists('test_key')
-        self.assertTrue(get_conn.called_once())
+        redis = hook.get_conn()
+
+        self.assertTrue(redis.ping(), 'Connection to Redis with PING works.')
+
+    def test_real_get_and_set(self):
+        hook = RedisHook(redis_conn_id='redis_default')
+        redis = hook.get_conn()
+
+        self.assertTrue(redis.set('test_key', 'test_value'), 'Connection to 
Redis with SET works.')
+        self.assertEqual(redis.get('test_key'), b'test_value', 'Connection to 
Redis with GET works.')
+        self.assertEqual(redis.delete('test_key'), 1, 'Connection to Redis 
with DELETE works.')
 
 
 if __name__ == '__main__':
diff --git a/tests/contrib/sensors/test_redis_sensor.py 
b/tests/contrib/sensors/test_redis_sensor.py
index 394c8e574b..ec2868c6f7 100644
--- a/tests/contrib/sensors/test_redis_sensor.py
+++ b/tests/contrib/sensors/test_redis_sensor.py
@@ -19,11 +19,9 @@
 
 
 import unittest
-
-from mock import patch
-
 from airflow import DAG
 from airflow import configuration
+from airflow.contrib.hooks.redis_hook import RedisHook
 from airflow.contrib.sensors.redis_key_sensor import RedisKeySensor
 from airflow.utils import timezone
 
@@ -47,22 +45,13 @@ def setUp(self):
             key='test_key'
         )
 
-    @patch("airflow.contrib.hooks.redis_hook.RedisHook.key_exists")
-    def test_poke(self, key_exists):
-        key_exists.return_value = True
-        self.assertTrue(self.sensor.poke(None))
-
-        key_exists.return_value = False
-        self.assertFalse(self.sensor.poke(None))
-
-    @patch("airflow.contrib.hooks.redis_hook.StrictRedis.exists")
-    def test_existing_key_called(self, redis_client_exists):
-        self.sensor.run(
-            start_date=DEFAULT_DATE,
-            end_date=DEFAULT_DATE, ignore_ti_state=True
-        )
-
-        self.assertTrue(redis_client_exists.called_with('test_key'))
+    def test_poke(self):
+        hook = RedisHook(redis_conn_id='redis_default')
+        redis = hook.get_conn()
+        redis.set('test_key', 'test_value')
+        self.assertTrue(self.sensor.poke(None), "Key exists on first call.")
+        redis.delete('test_key')
+        self.assertFalse(self.sensor.poke(None), "Key does NOT exists on 
second call.")
 
 
 if __name__ == '__main__':


 

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
[email protected]


> Fix for Redis Hook for not authorised connection calls.
> -------------------------------------------------------
>
>                 Key: AIRFLOW-3250
>                 URL: https://issues.apache.org/jira/browse/AIRFLOW-3250
>             Project: Apache Airflow
>          Issue Type: Bug
>          Components: db
>    Affects Versions: 1.9.0, 1.10.0, 2.0.0
>            Reporter: Pawel Graczyk
>            Assignee: Pawel Graczyk
>            Priority: Minor
>             Fix For: 1.9.0, 1.10.0
>
>
> Current implementation of AIRFLOW-999 needs fixes.
> 1. Password stays None and not 'None' (str) in case there is no password set, 
> otherwise AUTH call will be send and that produces errors on connection to 
> Redis that does not expect authorisation calls.
>  2. Reference to connection is set on get_conn hook object method rather than 
> on __init__
>  3. Trivial method key_exists of hook object removed
>  4. Fixes for unit tests so it deal with hook code and not related 
> dependencies such as Redis and DB connections. 



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to