Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id 9BECD200BE0 for ; Fri, 2 Dec 2016 20:30:11 +0100 (CET) Received: by cust-asf.ponee.io (Postfix) id 9AC09160B24; Fri, 2 Dec 2016 19:30:11 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id C4897160B08 for ; Fri, 2 Dec 2016 20:30:10 +0100 (CET) Received: (qmail 50243 invoked by uid 500); 2 Dec 2016 19:30:10 -0000 Mailing-List: contact commits-help@beam.incubator.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@beam.incubator.apache.org Delivered-To: mailing list commits@beam.incubator.apache.org Received: (qmail 50223 invoked by uid 99); 2 Dec 2016 19:30:10 -0000 Received: from pnap-us-west-generic-nat.apache.org (HELO spamd3-us-west.apache.org) (209.188.14.142) by apache.org (qpsmtpd/0.29) with ESMTP; Fri, 02 Dec 2016 19:30:10 +0000 Received: from localhost (localhost [127.0.0.1]) by spamd3-us-west.apache.org (ASF Mail Server at spamd3-us-west.apache.org) with ESMTP id 81AAC180286 for ; Fri, 2 Dec 2016 19:30:09 +0000 (UTC) X-Virus-Scanned: Debian amavisd-new at spamd3-us-west.apache.org X-Spam-Flag: NO X-Spam-Score: -6.219 X-Spam-Level: X-Spam-Status: No, score=-6.219 tagged_above=-999 required=6.31 tests=[KAM_ASCII_DIVIDERS=0.8, KAM_LAZY_DOMAIN_SECURITY=1, RCVD_IN_DNSWL_HI=-5, RCVD_IN_MSPIKE_H3=-0.01, RCVD_IN_MSPIKE_WL=-0.01, RP_MATCHES_RCVD=-2.999] autolearn=disabled Received: from mx1-lw-eu.apache.org ([10.40.0.8]) by localhost (spamd3-us-west.apache.org [10.40.0.10]) (amavisd-new, port 10024) with ESMTP id JtTB-neH2wx2 for ; Fri, 2 Dec 2016 19:30:07 +0000 (UTC) Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by mx1-lw-eu.apache.org (ASF Mail Server at mx1-lw-eu.apache.org) with SMTP id A770E5F369 for ; Fri, 2 Dec 2016 19:30:06 +0000 (UTC) Received: (qmail 50195 invoked by uid 99); 2 Dec 2016 19:30:05 -0000 Received: from git1-us-west.apache.org (HELO git1-us-west.apache.org) (140.211.11.23) by apache.org (qpsmtpd/0.29) with ESMTP; Fri, 02 Dec 2016 19:30:05 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id D6335E04BB; Fri, 2 Dec 2016 19:30:05 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: robertwb@apache.org To: commits@beam.incubator.apache.org Date: Fri, 02 Dec 2016 19:30:05 -0000 Message-Id: <5eba7d6a04034249afe6a75bf91857ef@git.apache.org> X-Mailer: ASF-Git Admin Mailer Subject: [1/2] incubator-beam git commit: auth: add application default credentials as fallback archived-at: Fri, 02 Dec 2016 19:30:11 -0000 Repository: incubator-beam Updated Branches: refs/heads/python-sdk 9ded359da -> a463f000e auth: add application default credentials as fallback Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/01bddf29 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/01bddf29 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/01bddf29 Branch: refs/heads/python-sdk Commit: 01bddf296dfb84a70bc733add6a76c76cf6afaef Parents: 9ded359 Author: Vikas Kedigehalli Authored: Wed Nov 30 17:55:20 2016 -0800 Committer: Robert Bradshaw Committed: Fri Dec 2 11:29:41 2016 -0800 ---------------------------------------------------------------------- sdks/python/apache_beam/internal/auth.py | 37 +++++++++++++++----- .../apache_beam/io/datastore/v1/datastoreio.py | 10 ++++-- .../apache_beam/io/datastore/v1/helper.py | 6 ++-- 3 files changed, 37 insertions(+), 16 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/01bddf29/sdks/python/apache_beam/internal/auth.py ---------------------------------------------------------------------- diff --git a/sdks/python/apache_beam/internal/auth.py b/sdks/python/apache_beam/internal/auth.py index f324a2d..a043fcf 100644 --- a/sdks/python/apache_beam/internal/auth.py +++ b/sdks/python/apache_beam/internal/auth.py @@ -24,7 +24,7 @@ import os import sys import urllib2 - +from oauth2client.client import GoogleCredentials from oauth2client.client import OAuth2Credentials from apache_beam.utils import processes @@ -125,6 +125,14 @@ def get_service_credentials(): # them again. return GCEMetadataCredentials(user_agent=user_agent) else: + client_scopes = [ + 'https://www.googleapis.com/auth/bigquery', + 'https://www.googleapis.com/auth/cloud-platform', + 'https://www.googleapis.com/auth/devstorage.full_control', + 'https://www.googleapis.com/auth/userinfo.email', + 'https://www.googleapis.com/auth/datastore' + ] + # We are currently being run from the command line. google_cloud_options = PipelineOptions( sys.argv).view_as(GoogleCloudOptions) @@ -135,13 +143,6 @@ def get_service_credentials(): if not os.path.exists(google_cloud_options.service_account_key_file): raise AuthenticationException( 'Specified service account key file does not exist.') - client_scopes = [ - 'https://www.googleapis.com/auth/bigquery', - 'https://www.googleapis.com/auth/cloud-platform', - 'https://www.googleapis.com/auth/devstorage.full_control', - 'https://www.googleapis.com/auth/userinfo.email', - 'https://www.googleapis.com/auth/datastore' - ] # The following code uses oauth2client >=2.0.0 functionality and if this # is not available due to import errors will use 1.5.2 functionality. @@ -163,4 +164,22 @@ def get_service_credentials(): user_agent=user_agent) else: - return _GCloudWrapperCredentials(user_agent) + try: + credentials = _GCloudWrapperCredentials(user_agent) + # Check if we are able to get an access token. If not fallback to + # application default credentials. + credentials.get_access_token() + return credentials + except AuthenticationException: + logging.warning('Unable to find credentials from gcloud.') + + # Falling back to application default credentials. + try: + credentials = GoogleCredentials.get_application_default() + credentials = credentials.create_scoped(client_scopes) + logging.debug('Connecting using Google Application Default ' + 'Credentials.') + return credentials + except Exception: + logging.warning('Unable to find default credentials to use.') + raise http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/01bddf29/sdks/python/apache_beam/io/datastore/v1/datastoreio.py ---------------------------------------------------------------------- diff --git a/sdks/python/apache_beam/io/datastore/v1/datastoreio.py b/sdks/python/apache_beam/io/datastore/v1/datastoreio.py index 20466b9..054002f 100644 --- a/sdks/python/apache_beam/io/datastore/v1/datastoreio.py +++ b/sdks/python/apache_beam/io/datastore/v1/datastoreio.py @@ -22,6 +22,7 @@ import logging from google.datastore.v1 import datastore_pb2 from googledatastore import helper as datastore_helper +from apache_beam.internal import auth from apache_beam.io.datastore.v1 import helper from apache_beam.io.datastore.v1 import query_splitter from apache_beam.transforms import Create @@ -153,7 +154,8 @@ class ReadFromDatastore(PTransform): self._num_splits = num_splits def start_bundle(self, context): - self._datastore = helper.get_datastore(self._project) + self._datastore = helper.get_datastore(self._project, + auth.get_service_credentials()) def process(self, p_context, *args, **kwargs): # distinct key to be used to group query splits. @@ -208,7 +210,8 @@ class ReadFromDatastore(PTransform): self._datastore = None def start_bundle(self, context): - self._datastore = helper.get_datastore(self._project) + self._datastore = helper.get_datastore(self._project, + auth.get_service_credentials()) def process(self, p_context, *args, **kwargs): query = p_context.element @@ -338,7 +341,8 @@ class _Mutate(PTransform): def start_bundle(self, context): self._mutations = [] - self._datastore = helper.get_datastore(self._project) + self._datastore = helper.get_datastore(self._project, + auth.get_service_credentials()) def process(self, context): self._mutations.append(context.element) http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/01bddf29/sdks/python/apache_beam/io/datastore/v1/helper.py ---------------------------------------------------------------------- diff --git a/sdks/python/apache_beam/io/datastore/v1/helper.py b/sdks/python/apache_beam/io/datastore/v1/helper.py index 28cb123..720f30a 100644 --- a/sdks/python/apache_beam/io/datastore/v1/helper.py +++ b/sdks/python/apache_beam/io/datastore/v1/helper.py @@ -95,11 +95,9 @@ def str_compare(s1, s2): return 1 -def get_datastore(project): +def get_datastore(project, credentials): """Returns a Cloud Datastore client.""" - credentials = datastore_helper.get_credentials_from_env() - datastore = Datastore(project, credentials) - return datastore + return Datastore(project, credentials) def make_request(project, namespace, query):