You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hawq.apache.org by rl...@apache.org on 2017/07/24 08:53:20 UTC

[2/4] incubator-hawq git commit: HAWQ-1502. Support TDE write function.

HAWQ-1502. Support TDE write function.


Project: http://git-wip-us.apache.org/repos/asf/incubator-hawq/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-hawq/commit/0d6a7440
Tree: http://git-wip-us.apache.org/repos/asf/incubator-hawq/tree/0d6a7440
Diff: http://git-wip-us.apache.org/repos/asf/incubator-hawq/diff/0d6a7440

Branch: refs/heads/master
Commit: 0d6a74406a4599f70b5a6cc73554d647394de382
Parents: c024842
Author: amyrazz44 <ab...@pivotal.io>
Authored: Tue Jul 11 15:57:26 2017 +0800
Committer: rlei <rl...@pivotal.io>
Committed: Mon Jul 24 16:53:15 2017 +0800

----------------------------------------------------------------------
 depends/libhdfs3/mock/MockCryptoCodec.h         |  37 +++
 depends/libhdfs3/mock/MockHttpClient.h          |  29 +-
 depends/libhdfs3/mock/MockKmsClientProvider.h   |  12 +-
 depends/libhdfs3/src/client/CryptoCodec.cpp     | 178 ++++++++++
 depends/libhdfs3/src/client/CryptoCodec.h       |  88 +++++
 depends/libhdfs3/src/client/HttpClient.cpp      | 254 +++++++-------
 depends/libhdfs3/src/client/HttpClient.h        | 220 ++++++-------
 .../libhdfs3/src/client/KmsClientProvider.cpp   | 327 ++++++++++---------
 depends/libhdfs3/src/client/KmsClientProvider.h | 166 +++++-----
 .../libhdfs3/src/client/OutputStreamImpl.cpp    |  47 ++-
 depends/libhdfs3/src/client/OutputStreamImpl.h  |  26 ++
 depends/libhdfs3/src/client/UserInfo.h          |   8 +-
 depends/libhdfs3/src/common/SessionConfig.cpp   |  17 +-
 depends/libhdfs3/src/common/SessionConfig.h     |  24 +-
 depends/libhdfs3/test/data/function-test.xml    |  15 +
 .../libhdfs3/test/function/TestCInterface.cpp   | 150 ++++++++-
 .../libhdfs3/test/function/TestKmsClient.cpp    | 173 +++++-----
 .../libhdfs3/test/function/TestOutputStream.cpp |   2 +-
 .../libhdfs3/test/unit/UnitTestCryptoCodec.cpp  | 133 ++++++++
 .../libhdfs3/test/unit/UnitTestOutputStream.cpp |  65 +++-
 20 files changed, 1359 insertions(+), 612 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/mock/MockCryptoCodec.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/mock/MockCryptoCodec.h b/depends/libhdfs3/mock/MockCryptoCodec.h
new file mode 100644
index 0000000..4d23e11
--- /dev/null
+++ b/depends/libhdfs3/mock/MockCryptoCodec.h
@@ -0,0 +1,37 @@
+/********************************************************************
+ * 2014 - 
+ * open source under Apache License Version 2.0
+ ********************************************************************/
+/**
+ * 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.
+ */
+#ifndef _HDFS_LIBHDFS3_MOCK_CRYPTOCODEC_H_
+#define _HDFS_LIBHDFS3_MOCK_CRYPTOCODEC_H_
+
+#include "gmock/gmock.h"
+
+#include "client/CryptoCodec.h"
+#include "client/KmsClientProvider.h"
+
+class MockCryptoCodec: public Hdfs::CryptoCodec {
+public:
+  MockCryptoCodec(FileEncryptionInfo *encryptionInfo, shared_ptr<KmsClientProvider> kcp, int32_t bufSize) : CryptoCodec(encryptionInfo, kcp, bufSize) {}
+  MOCK_METHOD2(encode, std::string(const char * buffer,int64_t size));
+  MOCK_METHOD2(decode, std::string(const char * buffer,int64_t size));
+};
+
+#endif /* _HDFS_LIBHDFS3_MOCK_CRYPTOCODEC_H_ */

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/mock/MockHttpClient.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/mock/MockHttpClient.h b/depends/libhdfs3/mock/MockHttpClient.h
index d0e1fd4..9da1186 100644
--- a/depends/libhdfs3/mock/MockHttpClient.h
+++ b/depends/libhdfs3/mock/MockHttpClient.h
@@ -32,21 +32,20 @@ using boost::property_tree::ptree;
 
 class MockHttpClient: public Hdfs::HttpClient {
 public:
-  MOCK_METHOD0(post, std::string());
-  MOCK_METHOD0(del, std::string());
-  MOCK_METHOD0(put, std::string());
-  MOCK_METHOD0(get, std::string());
-
-  std::string getPostResult(FileEncryptionInfo &encryptionInfo) {
-	ptree map;
-	map.put("name", encryptionInfo.getKeyName());
-	map.put("iv", encryptionInfo.getIv());
-	map.put("material", encryptionInfo.getKey());
-
-	std::string json = KmsClientProvider::toJson(map);
-	return json;
-  }
-
+    MOCK_METHOD0(post, std::string());
+    MOCK_METHOD0(del, std::string());
+    MOCK_METHOD0(put, std::string());
+    MOCK_METHOD0(get, std::string());
+
+    std::string getPostResult(FileEncryptionInfo &encryptionInfo) {
+        ptree map;
+        map.put("name", encryptionInfo.getKeyName());
+        map.put("iv", encryptionInfo.getIv());
+        map.put("material", encryptionInfo.getKey());
+
+        std::string json = KmsClientProvider::toJson(map);
+        return json;
+    }
 
 };
 

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/mock/MockKmsClientProvider.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/mock/MockKmsClientProvider.h b/depends/libhdfs3/mock/MockKmsClientProvider.h
index e530230..81fb8f3 100644
--- a/depends/libhdfs3/mock/MockKmsClientProvider.h
+++ b/depends/libhdfs3/mock/MockKmsClientProvider.h
@@ -30,12 +30,12 @@ using namespace Hdfs::Internal;
 
 class MockKmsClientProvider: public Hdfs::KmsClientProvider {
 public:
-  MockKmsClientProvider(std::shared_ptr<RpcAuth> auth, std::shared_ptr<SessionConfig> conf) : KmsClientProvider(auth, conf) {}
-  MOCK_METHOD1(setHttpClient, void(std::shared_ptr<HttpClient> hc));
-  MOCK_METHOD1(getKeyMetadata, ptree(const FileEncryptionInfo &encryptionInfo));
-  MOCK_METHOD1(deleteKey, void(const FileEncryptionInfo &encryptionInfo));
-  MOCK_METHOD1(decryptEncryptedKey, ptree(const FileEncryptionInfo &encryptionInfo));
-  MOCK_METHOD5(createKey, void(const std::string &keyName, const std::string &cipher, const int length, const std::string &material, const std::string &description));
+    MockKmsClientProvider(shared_ptr<RpcAuth> auth, shared_ptr<SessionConfig> conf) : KmsClientProvider(auth, conf) {}
+    MOCK_METHOD1(setHttpClient, void(shared_ptr<HttpClient> hc));
+    MOCK_METHOD1(getKeyMetadata, ptree(const FileEncryptionInfo &encryptionInfo));
+    MOCK_METHOD1(deleteKey, void(const FileEncryptionInfo &encryptionInfo));
+    MOCK_METHOD1(decryptEncryptedKey, ptree(const FileEncryptionInfo &encryptionInfo));
+    MOCK_METHOD5(createKey, void(const std::string &keyName, const std::string &cipher, const int length, const std::string &material, const std::string &description));
 
  ptree getEDKResult(FileEncryptionInfo &encryptionInfo) {
     ptree map;

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/CryptoCodec.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/CryptoCodec.cpp b/depends/libhdfs3/src/client/CryptoCodec.cpp
new file mode 100644
index 0000000..6ba1b74
--- /dev/null
+++ b/depends/libhdfs3/src/client/CryptoCodec.cpp
@@ -0,0 +1,178 @@
+/********************************************************************
+ * 2014 -
+ * open source under Apache License Version 2.0
+ ********************************************************************/
+/**
+ * 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.
+ */
+
+#include "CryptoCodec.h"
+#include "Logger.h"
+
+using namespace Hdfs::Internal;
+
+namespace Hdfs {
+
+/**
+ * Construct a CryptoCodec instance.
+ * @param encryptionInfo the encryption info of file.
+ * @param kcp a KmsClientProvider instance to get key from kms server.
+ * @param bufSize crypto buffer size.
+ */
+CryptoCodec::CryptoCodec(FileEncryptionInfo *encryptionInfo, shared_ptr<KmsClientProvider> kcp, int32_t bufSize) : encryptionInfo(encryptionInfo), kcp(kcp), bufSize(bufSize)
+{
+
+    /* Init global status. */
+    ERR_load_crypto_strings();
+    OpenSSL_add_all_algorithms();
+    OPENSSL_config(NULL);
+
+    /* Create cipher context. */
+    encryptCtx = EVP_CIPHER_CTX_new();
+    cipher = NULL;
+
+}
+
+/**
+ * Destroy a CryptoCodec instance.
+ */
+CryptoCodec::~CryptoCodec()
+{
+    if (encryptCtx)
+        EVP_CIPHER_CTX_free(encryptCtx);
+}
+
+/**
+ * Get decrypted key from kms.
+ */
+std::string CryptoCodec::getDecryptedKeyFromKms()
+{
+    ptree map = kcp->decryptEncryptedKey(*encryptionInfo);
+    std::string key;
+    try {
+        key = map.get < std::string > ("material");
+    } catch (...) {
+        THROW(HdfsIOException, "CryptoCodec : Can not get key from kms.");
+    }
+
+    int rem = key.length() % 4;
+    if (rem) {
+        rem = 4 - rem;
+        while (rem != 0) {
+            key = key + "=";
+            rem--;
+        }
+    }
+
+    std::replace(key.begin(), key.end(), '-', '+');
+    std::replace(key.begin(), key.end(), '_', '/');
+
+    LOG(INFO, "CryptoCodec : getDecryptedKeyFromKms material is :%s", key.c_str());
+
+    key = KmsClientProvider::base64Decode(key);
+    return key;
+
+	
+}
+
+/**
+ * Common encode/decode buffer method.
+ * @param buffer the buffer to be encode/decode.
+ * @param size the size of buffer.
+ * @param enc true is for encode, false is for decode.
+ * @return return the encode/decode buffer.
+ */
+std::string CryptoCodec::endecInternal(const char * buffer, int64_t size, bool enc)
+{
+    std::string key = encryptionInfo->getKey();
+    std::string iv = encryptionInfo->getIv();
+    LOG(INFO,
+            "CryptoCodec : endecInternal info. key:%s, iv:%s, buffer:%s, size:%d, is_encode:%d.",
+            key.c_str(), iv.c_str(), buffer, size, enc);
+	
+    /* Get decrypted key from KMS */
+    key = getDecryptedKeyFromKms();
+
+    /* Select cipher method based on the key length. */
+    if (key.length() == KEY_LENGTH_256) {
+        cipher = EVP_aes_256_ctr();
+    } else if (key.length() == KEY_LENGTH_128) {
+        cipher = EVP_aes_128_ctr();
+    } else {
+        THROW(InvalidParameter, "CryptoCodec : Invalid key length.");
+    }
+
+    /* Init cipher context with cipher method, encrypted key and IV from KMS. */
+    int encode = enc ? 1 : 0;
+    if (!EVP_CipherInit_ex(encryptCtx, cipher, NULL,
+            (const unsigned char *) key.c_str(),
+            (const unsigned char *) iv.c_str(), encode)) {
+        LOG(WARNING, "EVP_CipherInit_ex failed");
+    }
+    LOG(DEBUG3, "EVP_CipherInit_ex successfully");
+    EVP_CIPHER_CTX_set_padding(encryptCtx, 0);
+
+    /* Encode/decode buffer within cipher context. */
+    std::string result;
+    result.resize(size);
+    int offset = 0;
+    int remaining = size;
+    int len = 0;
+    /* If the encode/decode buffer size larger than crypto buffer size, encode/decode buffer one by one. */
+    while (remaining > bufSize) {
+        if (!EVP_CipherUpdate(encryptCtx, (unsigned char *) &result[offset],
+                &len, (const unsigned char *) buffer + offset, bufSize)) {
+            std::string err = ERR_lib_error_string(ERR_get_error());
+            THROW(HdfsIOException, "CryptoCodec : Cannot encrypt AES data %s",
+                    err.c_str());
+        }
+        offset += len;
+        remaining -= len;
+        LOG(DEBUG3,
+                "CryptoCodec : EVP_CipherUpdate successfully, result:%s, len:%d",
+                result.c_str(), len);
+    }
+    if (remaining) {
+        if (!EVP_CipherUpdate(encryptCtx, (unsigned char *) &result[offset],
+                &len, (const unsigned char *) buffer + offset, remaining)) {
+            std::string err = ERR_lib_error_string(ERR_get_error());
+            THROW(HdfsIOException, "CryptoCodec : Cannot encrypt AES data %s",
+                    err.c_str());
+        }
+    }
+
+    return result;
+}
+
+/**
+ * Encode buffer.
+ */
+std::string CryptoCodec::encode(const char * buffer, int64_t size)
+{
+    return endecInternal(buffer, size, true);
+}
+
+/**
+ * Decode buffer.
+ */	
+std::string CryptoCodec::decode(const char * buffer, int64_t size)
+{
+    return endecInternal(buffer, size, false);
+}
+
+}
+

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/CryptoCodec.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/CryptoCodec.h b/depends/libhdfs3/src/client/CryptoCodec.h
new file mode 100644
index 0000000..e45599b
--- /dev/null
+++ b/depends/libhdfs3/src/client/CryptoCodec.h
@@ -0,0 +1,88 @@
+/********************************************************************
+ * 2014 -
+ * open source under Apache License Version 2.0
+ ********************************************************************/
+/**
+ * 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.
+ */
+#ifndef _HDFS_LIBHDFS3_CLIENT_CRYPTOCODEC_H_
+#define _HDFS_LIBHDFS3_CLIENT_CRYPTOCODEC_H_
+
+#include <string>
+
+#include "openssl/conf.h"
+#include "openssl/evp.h"
+#include "openssl/err.h"
+#include "FileEncryptionInfo.h"
+#include "KmsClientProvider.h"
+
+#define KEY_LENGTH_256 32
+#define KEY_LENGTH_128 16
+
+namespace Hdfs {
+
+class CryptoCodec {
+public:
+    /**
+     * Construct a CryptoCodec instance.
+     * @param encryptionInfo the encryption info of file.
+     * @param kcp a KmsClientProvider instance to get key from kms server.
+     * @param bufSize crypto buffer size.
+     */
+    CryptoCodec(FileEncryptionInfo *encryptionInfo, shared_ptr<KmsClientProvider> kcp, int32_t bufSize);
+
+    /**
+     * Destroy a CryptoCodec instance.
+     */
+    virtual ~CryptoCodec();
+
+    /**
+     * Encode buffer.
+     */
+    virtual std::string encode(const char * buffer, int64_t size);
+
+    /**
+     * Decode buffer.
+     */
+    virtual std::string decode(const char * buffer, int64_t size);
+
+private:
+
+    /**
+     * Common encode/decode buffer method.
+     * @param buffer the buffer to be encode/decode.
+     * @param size the size of buffer.
+     * @param enc true is for encode, false is for decode.
+     * @return return the encode/decode buffer.
+     */
+    std::string endecInternal(const char *buffer, int64_t size, bool enc);
+
+    /**
+     * Get decrypted key from kms.
+     */
+    std::string getDecryptedKeyFromKms();
+
+    shared_ptr<KmsClientProvider> kcp;
+    FileEncryptionInfo *encryptionInfo;
+    EVP_CIPHER_CTX *encryptCtx;
+    EVP_CIPHER_CTX *decryptCtx;
+    const EVP_CIPHER *cipher;
+    int32_t bufSize;
+};
+
+}
+#endif

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/HttpClient.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/HttpClient.cpp b/depends/libhdfs3/src/client/HttpClient.cpp
index 6a80a99..562f599 100644
--- a/depends/libhdfs3/src/client/HttpClient.cpp
+++ b/depends/libhdfs3/src/client/HttpClient.cpp
@@ -46,7 +46,6 @@ namespace Hdfs {
         THROW(HdfsIOException, fmt, curl_easy_strerror(res), errorString().c_str()); \
     }
 
-
 #define CURL_GETOPT_ERROR2(handle, option, optarg, fmt) \
     res = curl_easy_getinfo(handle, option, optarg); \
     if (res != CURLE_OK) { \
@@ -57,7 +56,6 @@ namespace Hdfs {
     CURL_GETOPT_ERROR2(handle, CURLINFO_RESPONSE_CODE, code, fmt);
 
 HttpClient::HttpClient() : curl(NULL), list(NULL) {
-
 }
 
 /**
@@ -65,9 +63,9 @@ HttpClient::HttpClient() : curl(NULL), list(NULL) {
  * @param url a url which is the address to send the request to the corresponding http server.
  */
 HttpClient::HttpClient(const std::string &url) {
-	curl = NULL;
-	list = NULL;
-	this->url = url;
+    curl = NULL;
+    list = NULL;
+    this->url = url;
 }
 
 /**
@@ -75,16 +73,17 @@ HttpClient::HttpClient(const std::string &url) {
  */
 HttpClient::~HttpClient()
 {
-	destroy();
+    destroy();
 }
 
 /**
  * Receive error string from curl.
  */
 std::string HttpClient::errorString() {
-	if (strlen(errbuf) == 0)
-		return "";
-	return errbuf;
+    if (strlen(errbuf) == 0) {
+        return "";
+    }
+    return errbuf;
 }
 
 /**
@@ -93,26 +92,25 @@ std::string HttpClient::errorString() {
  */
 size_t HttpClient::CurlWriteMemoryCallback(void *contents, size_t size, size_t nmemb, void *userp)
 {
-      size_t realsize = size * nmemb;
-	  if (userp == NULL || contents == NULL) {
-		  return 0;
-	  }
-      ((std::string *)userp)->append((const char *)contents, realsize);
-	  LOG(DEBUG2, "HttpClient : Http response is : %s", ((std::string *)userp)->c_str());
-      return realsize;
+    size_t realsize = size * nmemb;
+    if (userp == NULL || contents == NULL) {
+        return 0;
+    }
+    ((std::string *) userp)->append((const char *) contents, realsize);
+    LOG(DEBUG3, "HttpClient : Http response is : %s", ((std::string * )userp)->c_str());
+    return realsize;
 }
 
 /**
  * Init curl handler and set curl options.
  */
 void HttpClient::init() {
-	if (!initialized)
-	{
-		initialized = true;
-		if (curl_global_init(CURL_GLOBAL_ALL)) {
-			THROW(HdfsIOException, "Cannot initialize curl client for KMS");
-		}
-	}
+    if (!initialized) {
+        initialized = true;
+        if (curl_global_init (CURL_GLOBAL_ALL)) {
+            THROW(HdfsIOException, "Cannot initialize curl client for KMS");
+        }
+    }
 
 	curl = curl_easy_init();
 	if (!curl) {
@@ -142,69 +140,72 @@ void HttpClient::init() {
     CURL_SETOPT_ERROR2(curl, CURLOPT_WRITEDATA, (void *)&response,
         "Cannot initialize body reader data in HttpClient: %s: %s");
 
+
     /* Some servers don't like requests that are made without a user-agent
-        field, so we provide one */
+     * field, so we provide one
+     */
     CURL_SETOPT_ERROR2(curl, CURLOPT_USERAGENT, "libcurl-agent/1.0",
         "Cannot initialize user agent in HttpClient: %s: %s");
-	list = NULL;
-
+    list = NULL;
 }
 
 /**
  * Do clean up for curl.
  */
 void HttpClient::destroy() {
-	if (curl) {
-		curl_easy_cleanup(curl);
-	}
-	if (list) {
-		curl_slist_free_all(list);
-	}
+    if (curl) {
+        curl_easy_cleanup(curl);
+        curl = NULL;
+    }
+    if (list) {
+        curl_slist_free_all(list);
+        list = NULL;
+    }
+    initialized = false;
 }
 
 /**
  * Set url for http client.
  */
 void HttpClient::setURL(const std::string &url) {
-	this->url = url;
+    this->url = url;
 }
 
 /**
  * Set retry times for http request which can be configured in config file.
  */
 void HttpClient::setRequestRetryTimes(int request_retry_times) {
-	if (request_retry_times < 0) {
-		THROW(InvalidParameter, "HttpClient : Invalid value for request_retry_times.");
-	}
-	this->request_retry_times = request_retry_times;
+    if (request_retry_times < 0) {
+        THROW(InvalidParameter, "HttpClient : Invalid value for request_retry_times.");
+    }
+    this->request_retry_times = request_retry_times;
 }
 
 /**
  * Set request timeout which can be configured in config file.
  */
 void HttpClient::setRequestTimeout(int64_t curl_timeout) {
-	if (curl_timeout < 0) {
-		THROW(InvalidParameter, "HttpClient : Invalid value for curl_timeout.");
-	}
-	this->curl_timeout = curl_timeout;
+    if (curl_timeout < 0) {
+        THROW(InvalidParameter, "HttpClient : Invalid value for curl_timeout.");
+    }
+    this->curl_timeout = curl_timeout;
 }
 
 /**
  * Set headers for http client.
  */
 void HttpClient::setHeaders(const std::vector<std::string> &headers) {
-	if (!headers.empty()) {
-		this->headers = headers;
-		for (std::string header : headers) {
-			list = curl_slist_append(list, header.c_str());
-			if (!list) {
-				THROW(HdfsIOException, "Cannot add header in HttpClient.");
-			}
-		}	
-	}
-	else {
-		LOG(DEBUG1, "HttpClient : Header is empty.");				
-	}
+    if (!headers.empty()) {
+        this->headers = headers;
+        for (std::string header : headers) {
+            list = curl_slist_append(list, header.c_str());
+            if (!list) {
+                THROW(HdfsIOException, "Cannot add header in HttpClient.");
+            }
+        }
+    } else {
+        LOG(DEBUG3, "HttpClient : Header is empty.");
+    }
 }
 
 
@@ -212,14 +213,14 @@ void HttpClient::setHeaders(const std::vector<std::string> &headers) {
  * Set body for http client.
  */
 void HttpClient::setBody(const std::string &body) {
-	this->body = body;
+    this->body = body;
 }
 
 /**
  * Set expected response code.
  */
 void HttpClient::setExpectedResponseCode(int64_t response_code_ok) {
-	this->response_code_ok = response_code_ok;
+    this->response_code_ok = response_code_ok;
 }
 
 /**
@@ -228,107 +229,116 @@ void HttpClient::setExpectedResponseCode(int64_t response_code_ok) {
  * @return return response info.
  */
 std::string HttpClient::httpCommon(httpMethod method) {
-	
     /* Set headers and url. */
-	if (list != NULL) {
-		CURL_SETOPT_ERROR2(curl, CURLOPT_HTTPHEADER, list,
-        	        "Cannot initialize headers in HttpClient: %s: %s");
-	} else {
-		LOG(DEBUG1, "HttpClient : Http Header is NULL");
-	}
+    if (list != NULL) {
+        CURL_SETOPT_ERROR2(curl, CURLOPT_HTTPHEADER, list,
+                "Cannot initialize headers in HttpClient: %s: %s");
+    } else {
+        LOG(DEBUG3, "HttpClient : Http Header is NULL");
+    }
 
-	if (curl != NULL) {
-		CURL_SETOPT_ERROR2(curl, CURLOPT_URL, url.c_str(),
-    	        "Cannot initialize url in HttpClient: %s: %s");
-	} else {
-		LOG(DEBUG1, "HttpClient : Http URL is NULL");
-	}
+    if (curl != NULL) {
+        CURL_SETOPT_ERROR2(curl, CURLOPT_URL, url.c_str(),
+                "Cannot initialize url in HttpClient: %s: %s");
+    } else {
+        LOG(LOG_ERROR, "HttpClient : Http URL is NULL");
+    }
 
-	/* Set body based on different http method. */ 
-	switch(method) {
-		case HTTP_GET:
-			break;
-		case HTTP_POST:
-			CURL_SETOPT_ERROR2(curl, CURLOPT_COPYPOSTFIELDS, body.c_str(),
-                "Cannot initialize post data in HttpClient: %s: %s");
-			break;
-		case HTTP_DELETE:
-			CURL_SETOPT_ERROR2(curl, CURLOPT_CUSTOMREQUEST, "DELETE",
-                "Cannot initialize set customer request in HttpClient: %s: %s");
-			break;
-		case HTTP_PUT:
-			CURL_SETOPT_ERROR2(curl, CURLOPT_CUSTOMREQUEST, "PUT",
-                "Cannot initialize set customer request in HttpClient: %s: %s");
-			CURL_SETOPT_ERROR2(curl, CURLOPT_COPYPOSTFIELDS, body.c_str(),
-                "Cannot initialize post data in HttpClient: %s: %s");
-			break;
-	}
+    /* Set body based on different http method. */
+    switch (method) {
+        case HTTP_GET:
+        {
+            break;
+        }
+        case HTTP_POST:
+        {
+            CURL_SETOPT_ERROR2(curl, CURLOPT_COPYPOSTFIELDS, body.c_str(),
+                    "Cannot initialize post data in HttpClient: %s: %s");
+            break;
+        }
+        case HTTP_DELETE:
+        {
+            CURL_SETOPT_ERROR2(curl, CURLOPT_CUSTOMREQUEST, "DELETE",
+                    "Cannot initialize set customer request in HttpClient: %s: %s");
+            break;
+        }
+        case HTTP_PUT:
+        {
+            CURL_SETOPT_ERROR2(curl, CURLOPT_CUSTOMREQUEST, "PUT",
+                    "Cannot initialize set customer request in HttpClient: %s: %s");
+            CURL_SETOPT_ERROR2(curl, CURLOPT_COPYPOSTFIELDS, body.c_str(),
+                    "Cannot initialize post data in HttpClient: %s: %s");
+            break;
+        }
+        default:
+        {
+            LOG(LOG_ERROR, "HttpClient : unknown method: %d", method);
+        }
+    }
 
-	/* Do several http request try according to request_retry_times until got the right reponse code. */	
-	int64_t response_code = -1;	
-
-	while (request_retry_times >= 0 && response_code != response_code_ok) {
-		request_retry_times -= 1;
-		response = "";
-		CURL_SETOPT_ERROR2(curl, CURLOPT_TIMEOUT, curl_timeout, 
-					"Send request to http server timeout: %s: %s");
-		CURL_PERFORM(curl, "Could not send request in HttpClient: %s %s");
-		CURL_GET_RESPONSE(curl, &response_code,
-					"Cannot get response code in HttpClient: %s: %s");
-	}
-	LOG(DEBUG1, "HttpClient : The http method is %d. The http url is %s. The http response is %s.", method, url.c_str(), response.c_str());
-	return response;
+    /* Do several http request try according to request_retry_times
+     * until got the right response code.
+     */
+    int64_t response_code = -1;
+
+    while (request_retry_times >= 0 && response_code != response_code_ok) {
+        request_retry_times -= 1;
+        response = "";
+        CURL_SETOPT_ERROR2(curl, CURLOPT_TIMEOUT, curl_timeout,
+                "Send request to http server timeout: %s: %s");
+        CURL_PERFORM(curl, "Could not send request in HttpClient: %s %s");
+        CURL_GET_RESPONSE(curl, &response_code,
+                "Cannot get response code in HttpClient: %s: %s");
+    }
+    LOG(DEBUG3, "HttpClient : The http method is %d. The http url is %s. The http response is %s.",
+            method, url.c_str(), response.c_str());
+    return response;
 }
 
 /**
  * Http GET method.
  */
 std::string HttpClient::get() {
-	httpMethod method = HTTP_GET;
-	return httpCommon(method);
+    return httpCommon(HTTP_GET);
 }
 
 /**
  * Http POST method.
  */
 std::string HttpClient::post() {
-	httpMethod method = HTTP_POST;
-	return httpCommon(method);
+    return httpCommon(HTTP_POST);
 }
 
 /**
  * Http DELETE method.
  */
 std::string HttpClient::del() {
-	httpMethod method = HTTP_DELETE;
-	return httpCommon(method);
+    return httpCommon(HTTP_DELETE);
 }
 
 /**
  * Http PUT method.
  */
 std::string HttpClient::put() {
-	httpMethod method = HTTP_PUT;
-	return httpCommon(method);
+    return httpCommon(HTTP_PUT);
 }
 
 
 /**
  *  URL encodes the given string. 
- */	
+ */
 std::string HttpClient::escape(const std::string &data) {
-	if (curl) {
-		char *output = curl_easy_escape(curl, data.c_str(), data.length());
-		if (output) {
-			std::string out(output);
-			return out;
-		} else {
-			THROW(HdfsIOException, "HttpClient : Curl escape failed.");
-		}
-	} else {
-		LOG(WARNING, "HttpClient : Curl in escape method is NULL");
-	}
-
+    if (curl) {
+        char *output = curl_easy_escape(curl, data.c_str(), data.length());
+        if (output) {
+            std::string out(output);
+            return out;
+        } else {
+            THROW(HdfsIOException, "HttpClient : Curl escape failed.");
+        }
+    } else {
+        LOG(WARNING, "HttpClient : Curl in escape method is NULL");
+    }
 }
 }
 

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/HttpClient.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/HttpClient.h b/depends/libhdfs3/src/client/HttpClient.h
index 9bada33..c77789b 100644
--- a/depends/libhdfs3/src/client/HttpClient.h
+++ b/depends/libhdfs3/src/client/HttpClient.h
@@ -30,125 +30,125 @@
 
 typedef enum httpMethod {
     HTTP_GET = 0,
-	HTTP_POST = 1,
-	HTTP_DELETE = 2,
-	HTTP_PUT = 3
+    HTTP_POST = 1,
+    HTTP_DELETE = 2,
+    HTTP_PUT = 3
 } httpMethod;
 
 namespace Hdfs {
 
 class HttpClient {
 public:
-	HttpClient();
-
-	/**
-	 * Construct a HttpClient instance.
-	 * @param url a url which is the address to send the request to the corresponding http server. 
-	 */
-	HttpClient(const std::string &url);
-
-	/**
-	 * Destroy a HttpClient instance.
-	 */
-	virtual ~HttpClient();
-	
-	/**
-	 * Set url for http client.
-	 */
-	void setURL(const std::string &url);	
-
-	/**
-	 * Set headers for http client.
-	 */
-	void setHeaders(const std::vector<std::string> &headers);
-	
-	/**
-	 * Set body for http client.
-	 */
-	void setBody(const std::string &body);	
-	
-	/**
-	 * Set retry times for http request which can be configured in config file.
-	 */
-	void setRequestRetryTimes(int requst_retry_times);	
-	
-	/**
-	 * Set request timeout which can be configured in config file.
-	 */
-	void setRequestTimeout(int64_t curl_timeout);	
-	
-	/**
-	 * Set expected response code.
-	 */
-	void setExpectedResponseCode(int64_t response_code_ok);
-
-	/**
-	 * Init curl handler and set options for curl.
-	 */
-	void init();
-	
-	/**
-	 * Do clean up for curl.
-	 */
-	void destroy();	
-	
-	/**
-	 * Http POST method. 
-	 */
-	virtual std::string post();
-	
-	/**
-	 * Http DELETE method. 
-	 */
-	virtual std::string del();
-	
-	/**
-	 * Http PUT method. 
-	 */
-	virtual std::string put();
-	
-	/**
-	 * Http GET method. 
-	 */
-	virtual std::string get();
-
-	/**
-	 * URL encodes the given string. 
-	 */
-	std::string escape(const std::string &data);
-
-	/**
-	 * Receive error string from curl. 
-	 */
-	std::string errorString();
+    HttpClient();
+
+    /**
+     * Construct a HttpClient instance.
+     * @param url a url which is the address to send the request to the corresponding http server.
+     */
+    HttpClient(const std::string &url);
+
+    /**
+     * Destroy a HttpClient instance.
+     */
+    virtual ~HttpClient();
+
+    /**
+     * Set url for http client.
+     */
+    void setURL(const std::string &url);
+
+    /**
+     * Set headers for http client.
+     */
+    void setHeaders(const std::vector<std::string> &headers);
+
+    /**
+     * Set body for http client.
+     */
+    void setBody(const std::string &body);
+
+    /**
+     * Set retry times for http request which can be configured in config file.
+     */
+    void setRequestRetryTimes(int requst_retry_times);
+
+    /**
+     * Set request timeout which can be configured in config file.
+     */
+    void setRequestTimeout(int64_t curl_timeout);
+
+    /**
+     * Set expected response code.
+     */
+    void setExpectedResponseCode(int64_t response_code_ok);
+
+    /**
+     * Init curl handler and set options for curl.
+     */
+    void init();
+
+    /**
+     * Do clean up for curl.
+     */
+    void destroy();
+
+    /**
+     * Http POST method.
+     */
+    virtual std::string post();
+
+    /**
+     * Http DELETE method.
+     */
+    virtual std::string del();
+
+    /**
+     * Http PUT method.
+     */
+    virtual std::string put();
+
+    /**
+     * Http GET method.
+     */
+    virtual std::string get();
+
+    /**
+     * URL encodes the given string.
+     */
+    std::string escape(const std::string &data);
+
+    /**
+     * Receive error string from curl.
+     */
+    std::string errorString();
 
 private:
 
-	/**
- 	 * Http common method to get response info by sending request to http server.
-	 * @param method : define different http methods.
-	 * @return return response info.
-	 */
- 	std::string 	httpCommon(httpMethod method);
-
-	/**
-	 * Curl call back function to receive the reponse messages.
-	 * @return return the size of reponse messages.
-	 */
-	static size_t 	CurlWriteMemoryCallback(void *contents, size_t size, size_t nmemb, void *userp); 
-
-	static bool initialized;
-	CURLcode res;
-	std::string url;
-	std::vector<std::string> headers;
-	std::string body;
-	int64_t response_code_ok;
-	int request_retry_times;
-	int64_t curl_timeout;	
-	CURL *curl;
-	struct curl_slist *list;
-	std::string response;
-	char errbuf[CURL_ERROR_SIZE] = {0};
+    /**
+     * Http common method to get response info by sending request to http server.
+     * @param method : define different http methods.
+     * @return return response info.
+     */
+    std::string httpCommon(httpMethod method);
+
+    /**
+     * Curl call back function to receive the reponse messages.
+     * @return return the size of reponse messages.
+     */
+    static size_t CurlWriteMemoryCallback(void *contents, size_t size, size_t nmemb, void *userp);
+
+    static bool initialized;
+    CURLcode res;
+    std::string url;
+    std::vector<std::string> headers;
+    std::string body;
+    int64_t response_code_ok;
+    int request_retry_times;
+    int64_t curl_timeout;
+    CURL *curl;
+    struct curl_slist *list;
+    std::string response;
+    char errbuf[CURL_ERROR_SIZE] = { 0 };
 };
 
 }

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/KmsClientProvider.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/KmsClientProvider.cpp b/depends/libhdfs3/src/client/KmsClientProvider.cpp
index 596b07f..f1b4628 100644
--- a/depends/libhdfs3/src/client/KmsClientProvider.cpp
+++ b/depends/libhdfs3/src/client/KmsClientProvider.cpp
@@ -34,87 +34,85 @@ namespace Hdfs {
 /**
  * Convert ptree format to json format
  */
-std::string KmsClientProvider::toJson(const ptree &data)
-{
-	std::ostringstream buf;
-	try {
-		write_json(buf, data, false);
-		std::string json = buf.str();
-		return json;
-	} catch (...) {
-		THROW(HdfsIOException, "KmsClientProvider : Write json failed.");
-	}	
+std::string KmsClientProvider::toJson(const ptree &data) {
+    std::ostringstream buf;
+    try {
+        write_json(buf, data, false);
+        std::string json = buf.str();
+        return json;
+    } catch (...) {
+        THROW(HdfsIOException, "KmsClientProvider : Write json failed.");
+    }
 }
 
 /**
  * Convert json format to ptree format
  */
-ptree KmsClientProvider::fromJson(const std::string &data)
-{
-	ptree pt2;
-	try {
-		std::istringstream is(data);
-		read_json(is, pt2);
-		return pt2;
-	} catch (...) {
-		THROW(HdfsIOException, "KmsClientProvider : Read json failed.");
-	}
+ptree KmsClientProvider::fromJson(const std::string &data) {
+    ptree pt2;
+    try {
+        std::istringstream is(data);
+        read_json(is, pt2);
+        return pt2;
+    } catch (...) {
+        THROW(HdfsIOException, "KmsClientProvider : Read json failed.");
+    }
 }
 
 /**
  * Encode string to base64. 
  */
-std::string	KmsClientProvider::base64Encode(const std::string &data)
-{
-	char * buffer = NULL;
-	size_t len = 0;
-	int rc = 0;
-	std::string result;
+std::string KmsClientProvider::base64Encode(const std::string &data) {
+    char * buffer = NULL;
+    size_t len = 0;
+    int rc = 0;
+    std::string result;
 
-	LOG(DEBUG1, "KmsClientProvider : Encode data is %s", data.c_str());
+    LOG(DEBUG3, "KmsClientProvider : Encode data is %s", data.c_str());
 
-	if (GSASL_OK != (rc = gsasl_base64_to(data.c_str(), data.size(), &buffer, &len))) {
-		assert(GSASL_MALLOC_ERROR == rc);
+    if (GSASL_OK != (rc = gsasl_base64_to(data.data(), data.size(), &buffer, &len))) {
+        assert(GSASL_MALLOC_ERROR == rc);
         throw std::bad_alloc();
-	}
+    }
 
-	if (buffer) {
-		result.assign(buffer, len);
-		free(buffer);
-	} 
+    if (buffer) {
+        result.assign(buffer, len);
+        free(buffer);
+    }
 
-	if (!buffer || result.length() != len) {
-        THROW(HdfsIOException, "KmsClientProvider: Failed to encode string to base64");
+    if (!buffer || result.length() != len) {
+        THROW(HdfsIOException,
+                "KmsClientProvider: Failed to encode string to base64");
     }
 
-	return result;	
+    return result;
 }
 
 /**
  * Decode base64 to string.
  */
-std::string	KmsClientProvider::base64Decode(const std::string &data)
-{
-	char * buffer = NULL;
-	size_t len = 0;
-	int rc = 0;
-	std::string result;
-
-	if (GSASL_OK != (rc = gsasl_base64_from(data.c_str(), data.size(), &buffer, &len))) {
-		assert(GSASL_MALLOC_ERROR == rc);
+std::string KmsClientProvider::base64Decode(const std::string &data) {
+    char * buffer = NULL;
+    size_t len = 0;
+    int rc = 0;
+    std::string result;
+
+    if (GSASL_OK != (rc = gsasl_base64_from(data.data(), data.size(), &buffer, &len))) {
+        assert(GSASL_MALLOC_ERROR == rc);
         throw std::bad_alloc();
-	}
+    }
 
-	if (buffer) {
-		result.assign(buffer, len);
-		free(buffer);
-	} 
+    if (buffer) {
+        result.assign(buffer, len);
+        free(buffer);
+    }
 
-	if (!buffer || result.length() != len) {
-        THROW(HdfsIOException, "KmsClientProvider: Failed to decode base64 to string");
+    if (!buffer || result.length() != len) {
+        THROW(HdfsIOException,
+                "KmsClientProvider: Failed to decode base64 to string");
     }
 
-	return result;	
+    return result;
 }
 
 /**
@@ -122,18 +120,18 @@ std::string	KmsClientProvider::base64Decode(const std::string &data)
  * @param auth RpcAuth to get the auth method and user info.
  * @param conf a SessionConfig to get the configuration.
  */
-KmsClientProvider::KmsClientProvider(std::shared_ptr<RpcAuth> rpcAuth, std::shared_ptr<SessionConfig> config) : auth(rpcAuth), conf(config)
+KmsClientProvider::KmsClientProvider(shared_ptr<RpcAuth> rpcAuth, shared_ptr<SessionConfig> config) : auth(rpcAuth), conf(config)
 {
-	hc.reset(new HttpClient());
-	method = RpcAuth::ParseMethod(conf->getKmsMethod());
+    hc.reset(new HttpClient());
+    method = RpcAuth::ParseMethod(conf->getKmsMethod());
 }
 
 /**
  * Set HttpClient object.
  */
-void KmsClientProvider::setHttpClient(std::shared_ptr<HttpClient> hc)
+void KmsClientProvider::setHttpClient(shared_ptr<HttpClient> hc)
 {
-	this->hc = hc;
+    this->hc = hc;
 }
 
 /**
@@ -141,23 +139,21 @@ void KmsClientProvider::setHttpClient(std::shared_ptr<HttpClient> hc)
  */
 std::string KmsClientProvider::parseKmsUrl() 
 {
-	std::string start = "kms://";
+    std::string start = "kms://";
     std::string http = "http@";
     std::string https = "https@";
-	std::string urlParse = conf->getKmsUrl();
-	LOG(DEBUG2, "KmsClientProvider : Get kms url from conf : %s.", urlParse.c_str());
+    std::string urlParse = conf->getKmsUrl();
+    LOG(DEBUG3, "KmsClientProvider : Get kms url from conf : %s.",
+            urlParse.c_str());
     if (urlParse.compare(0, start.length(), start) == 0) {
         start = urlParse.substr(start.length());
         if (start.compare(0, http.length(), http) == 0) {
             return "http://" + start.substr(http.length());
-        }
-        else if (start.compare(0, https.length(), https) == 0) {
+        } else if (start.compare(0, https.length(), https) == 0) {
             return "https://" + start.substr(https.length());
-        }
-        else
+        } else
             THROW(HdfsIOException, "Bad KMS provider URL: %s", urlParse.c_str());
-    }
-    else
+    } else
         THROW(HdfsIOException, "Bad KMS provider URL: %s", urlParse.c_str());
 
 }
@@ -167,25 +163,27 @@ std::string KmsClientProvider::parseKmsUrl()
  */
 std::string KmsClientProvider::buildKmsUrl(const std::string &url, const std::string &urlSuffix)
 {
-		std::string baseUrl = url;
-        baseUrl = url + "/v1/" + urlSuffix;
-		std::size_t found = urlSuffix.find('?');
-
-        if (method == AuthMethod::KERBEROS) {
-            // todo
-			THROW(InvalidParameter, "KmsClientProvider : Not support kerberos yet.");
-        } else if (method == AuthMethod::SIMPLE) {
-            std::string user = auth->getUser().getRealUser();
-			LOG(DEBUG1, "KmsClientProvider : Kms urlSuffix is : %s. Auth real user is : %s.", urlSuffix.c_str(), user.c_str());
-            if (user.length() == 0)
-                user = auth->getUser().getKrbName();
-			if (found != std::string::npos)
-            	return baseUrl + "&user.name=" + user;
-			else
-				return baseUrl + "?user.name=" + user;
-        } else {
-            return baseUrl;
-        }	
+    std::string baseUrl = url;
+    baseUrl = url + "/v1/" + urlSuffix;
+    std::size_t found = urlSuffix.find('?');
+
+    if (method == AuthMethod::KERBEROS) {
+        // todo
+        THROW(InvalidParameter, "KmsClientProvider : Not support kerberos yet.");
+    } else if (method == AuthMethod::SIMPLE) {
+        std::string user = auth->getUser().getRealUser();
+        LOG(DEBUG3,
+                "KmsClientProvider : Kms urlSuffix is : %s. Auth real user is : %s.",
+                urlSuffix.c_str(), user.c_str());
+        if (user.length() == 0)
+            user = auth->getUser().getKrbName();
+        if (found != std::string::npos)
+            return baseUrl + "&user.name=" + user;
+        else
+            return baseUrl + "?user.name=" + user;
+    } else {
+        return baseUrl;
+    }
 }
 
 /**
@@ -193,8 +191,8 @@ std::string KmsClientProvider::buildKmsUrl(const std::string &url, const std::st
  */
 void KmsClientProvider::setCommonHeaders(std::vector<std::string>& headers)
 {
-	headers.push_back("Content-Type: application/json");
- 	headers.push_back("Accept: *");
+    headers.push_back("Content-Type: application/json");
+    headers.push_back("Accept: *");
 }
 
 
@@ -208,31 +206,34 @@ void KmsClientProvider::setCommonHeaders(std::vector<std::string>& headers)
  */
 void KmsClientProvider::createKey(const std::string &keyName, const std::string &cipher, const int length, const std::string &material, const std::string &description)
 {
-	hc->init();
-	/* Prepare url for HttpClient.*/
-	url = parseKmsUrl(); 
-	std::string urlSuffix = "keys";
-	url = buildKmsUrl(url, urlSuffix);
-	/* Prepare headers for HttpClient.*/
-	std::vector<std::string> headers;
-	setCommonHeaders(headers);
-	/* Prepare body for HttpClient. */
-	ptree map;
+    hc->init();
+    /* Prepare url for HttpClient.*/
+    url = parseKmsUrl();
+    std::string urlSuffix = "keys";
+    url = buildKmsUrl(url, urlSuffix);
+    /* Prepare headers for HttpClient.*/
+    std::vector<std::string> headers;
+    setCommonHeaders(headers);
+    /* Prepare body for HttpClient. */
+    ptree map;
     map.put("name", keyName);
     map.put("cipher", cipher);
-	map.put("description", description);
-    std::string body = toJson(map);	
-	/* Set options for HttpClient to get response. */
-	hc->setURL(url);
-	hc->setHeaders(headers);
-	hc->setBody(body);
-	hc->setRequestRetryTimes(conf->getHttpRequestRetryTimes());
-	hc->setRequestTimeout(conf->getCurlTimeOut());
-	hc->setExpectedResponseCode(201);
-	std::string response = hc->post();
-
-	LOG(INFO, "KmsClientProvider::createKey : The key name, key cipher, key length, key material, description are : %s, %s, %s, %s, %s. The kms url is : %s . The kms body is : %s. The response of kms server is : %s ." , keyName.c_str(), cipher.c_str(), length, material.c_str(), description.c_str(), url.c_str(), body.c_str(), response.c_str());
-		
+    map.put("description", description);
+    std::string body = toJson(map);
+    /* Set options for HttpClient to get response. */
+    hc->setURL(url);
+    hc->setHeaders(headers);
+    hc->setBody(body);
+    hc->setRequestRetryTimes(conf->getHttpRequestRetryTimes());
+    hc->setRequestTimeout(conf->getCurlTimeOut());
+    hc->setExpectedResponseCode(201);
+    std::string response = hc->post();
+
+    LOG(INFO,
+            "KmsClientProvider::createKey : The key name, key cipher, key length, key material, description are : %s, %s, %d, %s, %s. The kms url is : %s . The kms body is : %s. The response of kms server is : %s .",
+            keyName.c_str(), cipher.c_str(), length, material.c_str(),
+            description.c_str(), url.c_str(), body.c_str(), response.c_str());
+
 } 
 
 /**
@@ -242,20 +243,22 @@ void KmsClientProvider::createKey(const std::string &keyName, const std::string
  */
 ptree KmsClientProvider::getKeyMetadata(const FileEncryptionInfo &encryptionInfo)
 {
-	hc->init();
-	url = parseKmsUrl(); 
-	std::string urlSuffix = "key/" + hc->escape(encryptionInfo.getKeyName()) + "/_metadata";
-	url = buildKmsUrl(url, urlSuffix);
-	
-	hc->setURL(url);
-	hc->setExpectedResponseCode(200);
-	hc->setRequestRetryTimes(conf->getHttpRequestRetryTimes());
-	hc->setRequestTimeout(conf->getCurlTimeOut());
-	std::string response = hc->get();
-	
-	LOG(INFO, "KmsClientProvider::getKeyMetadata : The kms url is : %s. The response of kms server is : %s ." , url.c_str(), response.c_str());
-
-	return fromJson(response);
+    hc->init();
+    url = parseKmsUrl();
+    std::string urlSuffix = "key/" + hc->escape(encryptionInfo.getKeyName()) + "/_metadata";
+    url = buildKmsUrl(url, urlSuffix);
+
+    hc->setURL(url);
+    hc->setExpectedResponseCode(200);
+    hc->setRequestRetryTimes(conf->getHttpRequestRetryTimes());
+    hc->setRequestTimeout(conf->getCurlTimeOut());
+    std::string response = hc->get();
+
+    LOG(INFO,
+            "KmsClientProvider::getKeyMetadata : The kms url is : %s. The response of kms server is : %s .",
+            url.c_str(), response.c_str());
+
+    return fromJson(response);
 
 }
 
@@ -265,18 +268,20 @@ ptree KmsClientProvider::getKeyMetadata(const FileEncryptionInfo &encryptionInfo
  */
 void KmsClientProvider::deleteKey(const FileEncryptionInfo &encryptionInfo)
 {
-	hc->init();
-	url = parseKmsUrl(); 
-	std::string urlSuffix = "key/" + hc->escape(encryptionInfo.getKeyName());
-	url = buildKmsUrl(url, urlSuffix);
-	
-    hc->setURL(url);
-	hc->setExpectedResponseCode(200);
-	hc->setRequestRetryTimes(conf->getHttpRequestRetryTimes());
-	hc->setRequestTimeout(conf->getCurlTimeOut());
-	std::string response = hc->del();
+    hc->init();
+    url = parseKmsUrl();
+    std::string urlSuffix = "key/" + hc->escape(encryptionInfo.getKeyName());
+    url = buildKmsUrl(url, urlSuffix);
 
-	LOG(INFO, "KmsClientProvider::deleteKey : The kms url is : %s. The response of kms server is : %s ." , url.c_str(), response.c_str());
+    hc->setURL(url);
+    hc->setExpectedResponseCode(200);
+    hc->setRequestRetryTimes(conf->getHttpRequestRetryTimes());
+    hc->setRequestTimeout(conf->getCurlTimeOut());
+    std::string response = hc->del();
+
+    LOG(INFO,
+            "KmsClientProvider::deleteKey : The kms url is : %s. The response of kms server is : %s .",
+            url.c_str(), response.c_str());
 }
 
 /**
@@ -286,32 +291,34 @@ void KmsClientProvider::deleteKey(const FileEncryptionInfo &encryptionInfo)
  */
 ptree KmsClientProvider::decryptEncryptedKey(const FileEncryptionInfo &encryptionInfo)
 {
-	hc->init();
-	/* Prepare HttpClient url. */
-	url = parseKmsUrl(); 
-	std::string urlSuffix = "keyversion/" + hc->escape(encryptionInfo.getEzKeyVersionName()) + "/_eek?eek_op=decrypt";
-	url = buildKmsUrl(url, urlSuffix);
-	/* Prepare HttpClient headers. */
-	std::vector<std::string> headers;
-	setCommonHeaders(headers);
-	/* Prepare HttpClient body with json format. */
-	ptree map;
+    hc->init();
+    /* Prepare HttpClient url. */
+    url = parseKmsUrl();
+    std::string urlSuffix = "keyversion/" + hc->escape(encryptionInfo.getEzKeyVersionName()) + "/_eek?eek_op=decrypt";
+    url = buildKmsUrl(url, urlSuffix);
+    /* Prepare HttpClient headers. */
+    std::vector<std::string> headers;
+    setCommonHeaders(headers);
+    /* Prepare HttpClient body with json format. */
+    ptree map;
     map.put("name", encryptionInfo.getKeyName());
     map.put("iv", base64Encode(encryptionInfo.getIv()));
     map.put("material", base64Encode(encryptionInfo.getKey()));
-    std::string body = toJson(map);	
-
-	/* Set options for HttpClient. */
-	hc->setURL(url);
-	hc->setHeaders(headers);
-	hc->setBody(body);
-	hc->setExpectedResponseCode(200);
-	hc->setRequestRetryTimes(conf->getHttpRequestRetryTimes());
-	hc->setRequestTimeout(conf->getCurlTimeOut());
-	std::string response = hc->post();
-
-	LOG(INFO, "KmsClientProvider::decryptEncryptedKey : The kms url is : %s . The kms body is : %s. The response of kms server is : %s ." , url.c_str(), body.c_str(), response.c_str());
-	return fromJson(response);
+    std::string body = toJson(map);
+
+    /* Set options for HttpClient. */
+    hc->setURL(url);
+    hc->setHeaders(headers);
+    hc->setBody(body);
+    hc->setExpectedResponseCode(200);
+    hc->setRequestRetryTimes(conf->getHttpRequestRetryTimes());
+    hc->setRequestTimeout(conf->getCurlTimeOut());
+    std::string response = hc->post();
+
+    LOG(INFO,
+            "KmsClientProvider::decryptEncryptedKey : The kms url is : %s . The kms body is : %s. The response of kms server is : %s .",
+            url.c_str(), body.c_str(), response.c_str());
+    return fromJson(response);
 }
 
 }

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/KmsClientProvider.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/KmsClientProvider.h b/depends/libhdfs3/src/client/KmsClientProvider.h
index 5eef428..a6c4336 100644
--- a/depends/libhdfs3/src/client/KmsClientProvider.h
+++ b/depends/libhdfs3/src/client/KmsClientProvider.h
@@ -33,7 +33,7 @@
 #include <vector>
 #include "common/SessionConfig.h"
 #include "rpc/RpcAuth.h"
-
+#include "common/Memory.h"
 #include <boost/property_tree/ptree.hpp>
 
 using boost::property_tree::ptree;
@@ -45,97 +45,97 @@ namespace Hdfs {
 class KmsClientProvider {
 public:
 
-	/**
+    /**
      * Construct a KmsClientProvider instance.
      * @param auth RpcAuth to get the auth method and user info.
-	 * @param conf a SessionConfig to get the configuration.
+     * @param conf a SessionConfig to get the configuration.
      */
-	KmsClientProvider(std::shared_ptr<RpcAuth> auth, std::shared_ptr<SessionConfig> conf);
+    KmsClientProvider(shared_ptr<RpcAuth> auth, shared_ptr<SessionConfig> conf);
 
-	/**
+    /**
      * Destroy a KmsClientProvider instance.
      */
-	virtual ~KmsClientProvider(){
-	}
-
-	/**
-	 * Set HttpClient object.
-	 */
-	void setHttpClient(std::shared_ptr<HttpClient> hc);
-
-	/**
-	 * Create an encryption key from kms.
-	 * @param keyName the name of this key.
-	 * @param cipher the ciphertext of this key. e.g. "AES/CTR/NoPadding" .
-	 * @param length the length of this key.
-	 * @param material will be encode to base64.
-	 * @param description key's info.
- 	*/	
-	virtual void createKey(const std::string &keyName, const std::string &cipher, const int length, const std::string &material, const std::string &description);
-
-	/**
-	 * Get key metadata based on encrypted file's key name.
-	 * @param encryptionInfo the encryption info of file.
- 	* @return return response info about key metadata from kms server.
-	 */
-	virtual ptree getKeyMetadata(const FileEncryptionInfo &encryptionInfo);
-
-	/**
- 	 * Delete an encryption key from kms.
-	 * @param encryptionInfo the encryption info of file.
-	 */
-	virtual void deleteKey(const FileEncryptionInfo &encryptionInfo);
-
-	/**
- 	 * Decrypt an encrypted key from kms.
-	 * @param encryptionInfo the encryption info of file.
-	 * @return return decrypted key.
-	 */
-	virtual ptree decryptEncryptedKey(const FileEncryptionInfo &encryptionInfo);
-
-	/**
-	 * Encode string to base64.
-	 */
-	static std::string	base64Encode(const std::string &data);
-
-	/**
-	 * Decode base64 to string.
-	 */	
-	static std::string	base64Decode(const std::string &data);
+    virtual ~KmsClientProvider() {
+    }
+
+    /**
+     * Set HttpClient object.
+     */
+    void setHttpClient(shared_ptr<HttpClient> hc);
+
+    /**
+     * Create an encryption key from kms.
+     * @param keyName the name of this key.
+     * @param cipher the ciphertext of this key. e.g. "AES/CTR/NoPadding" .
+     * @param length the length of this key.
+     * @param material will be encode to base64.
+     * @param description key's info.
+     */
+    virtual void createKey(const std::string &keyName, const std::string &cipher, const int length, const std::string &material, const std::string &description);
+
+    /**
+     * Get key metadata based on encrypted file's key name.
+     * @param encryptionInfo the encryption info of file.
+     * @return return response info about key metadata from kms server.
+     */
+    virtual ptree getKeyMetadata(const FileEncryptionInfo &encryptionInfo);
+
+    /**
+     * Delete an encryption key from kms.
+     * @param encryptionInfo the encryption info of file.
+     */
+    virtual void deleteKey(const FileEncryptionInfo &encryptionInfo);
+
+    /**
+     * Decrypt an encrypted key from kms.
+     * @param encryptionInfo the encryption info of file.
+     * @return return decrypted key.
+     */
+    virtual ptree decryptEncryptedKey(const FileEncryptionInfo &encryptionInfo);
+
+    /**
+     * Encode string to base64.
+     */
+    static std::string base64Encode(const std::string &data);
+
+    /**
+     * Decode base64 to string.
+     */
+    static std::string base64Decode(const std::string &data);
 
 private:
 
-	/**
-	 * Convert ptree format to json format.
-	 */
-	static std::string  toJson(const ptree &data);
-
-	/**
- 	 * Convert json format to ptree format.
-	 */
-	static ptree		fromJson(const std::string &data);
-
-	/**
-	 * Parse kms url from configure file.
-	 */
-	std::string 		parseKmsUrl();
-
-	/**
-	 * Build kms url based on urlSuffix and different auth method.
-	 */
-	std::string 		buildKmsUrl(const std::string &url, const std::string &urlSuffix);
-	/**
- 	 * Set common headers for kms API.
- 	 */
-	void setCommonHeaders(std::vector<std::string>& headers);	
-
-	std::shared_ptr<HttpClient> 	hc;
-	std::string						url;
-
-	std::shared_ptr<RpcAuth> 		auth;
-	AuthMethod						method;
-	std::shared_ptr<SessionConfig> 	conf;
-	
+    /**
+     * Convert ptree format to json format.
+     */
+    static std::string toJson(const ptree &data);
+
+    /**
+     * Convert json format to ptree format.
+     */
+    static ptree fromJson(const std::string &data);
+
+    /**
+     * Parse kms url from configure file.
+     */
+    std::string parseKmsUrl();
+
+    /**
+     * Build kms url based on urlSuffix and different auth method.
+     */
+    std::string buildKmsUrl(const std::string &url, const std::string &urlSuffix);
+    /**
+     * Set common headers for kms API.
+     */
+    void setCommonHeaders(std::vector<std::string>& headers);
+
+    shared_ptr<HttpClient> hc;
+    std::string url;
+
+    shared_ptr<RpcAuth> auth;
+    AuthMethod method;
+    shared_ptr<SessionConfig> conf;
+
 };
 
 }

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/OutputStreamImpl.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/OutputStreamImpl.cpp b/depends/libhdfs3/src/client/OutputStreamImpl.cpp
index 340a4eb..4c5f869 100644
--- a/depends/libhdfs3/src/client/OutputStreamImpl.cpp
+++ b/depends/libhdfs3/src/client/OutputStreamImpl.cpp
@@ -43,7 +43,7 @@ OutputStreamImpl::OutputStreamImpl() :
 /*heartBeatStop(true),*/ closed(true), isAppend(false), syncBlock(false), checksumSize(0), chunkSize(
         0), chunksPerPacket(0), closeTimeout(0), heartBeatInterval(0), packetSize(0), position(
             0), replication(0), blockSize(0), bytesWritten(0), cursor(0), lastFlushed(
-                0), nextSeqNo(0), packets(0) {
+                0), nextSeqNo(0), packets(0), cryptoCodec(NULL), kcp(NULL) {
     if (HWCrc32c::available()) {
         checksum = shared_ptr < Checksum > (new HWCrc32c());
     } else {
@@ -86,6 +86,21 @@ void OutputStreamImpl::setError(const exception_ptr & error) {
     }
 }
 
+shared_ptr<CryptoCodec> OutputStreamImpl::getCryptoCodec() {
+    return cryptoCodec;
+}
+
+void OutputStreamImpl::setCryptoCodec(shared_ptr<CryptoCodec> cryptoCodec) {
+    this->cryptoCodec = cryptoCodec;
+}
+
+shared_ptr<KmsClientProvider> OutputStreamImpl::getKmsClientProvider() {
+    return kcp;
+}
+
+void OutputStreamImpl::setKmsClientProvider(shared_ptr<KmsClientProvider> kcp) {
+    this->kcp = kcp;
+}
 /**
  * To create or append a file.
  * @param fs hdfs file system.
@@ -236,6 +251,15 @@ void OutputStreamImpl::openInternal(shared_ptr<FileSystemInter> fs, const char *
 
     try {
         if (flag & Append) {
+            fileStatus = fs->getFileStatus(this->path.c_str());
+            FileEncryptionInfo *fileEnInfo = fileStatus.getFileEncryption();
+            if (fileStatus.isFileEncrypted()) {
+                if (cryptoCodec == NULL) {
+                    auth = shared_ptr<RpcAuth> (new RpcAuth(fs->getUserInfo(), RpcAuth::ParseMethod(conf->getKmsMethod())));
+                    kcp = shared_ptr<KmsClientProvider> (new KmsClientProvider(auth, conf));
+                    cryptoCodec = shared_ptr<CryptoCodec> (new CryptoCodec(fileEnInfo, kcp, conf->getCryptoBufferSize()));
+                }
+            }
             initAppend();
             LeaseRenewer::GetLeaseRenewer().StartRenew(filesystem);
             return;
@@ -248,7 +272,21 @@ void OutputStreamImpl::openInternal(shared_ptr<FileSystemInter> fs, const char *
 
     assert((flag & Create) || (flag & Overwrite));
     fs->create(this->path, permission, flag, createParent, this->replication,
-               this->blockSize);
+            this->blockSize);
+    fileStatus = fs->getFileStatus(this->path.c_str());
+    FileEncryptionInfo *fileEnInfo = fileStatus.getFileEncryption();
+    if (fileStatus.isFileEncrypted()) {
+        if (cryptoCodec == NULL) {
+            auth = shared_ptr<RpcAuth>(
+                    new RpcAuth(fs->getUserInfo(),
+                            RpcAuth::ParseMethod(conf->getKmsMethod())));
+            kcp = shared_ptr<KmsClientProvider>(
+                    new KmsClientProvider(auth, conf));
+            cryptoCodec = shared_ptr<CryptoCodec>(
+                    new CryptoCodec(fileEnInfo, kcp,
+                            conf->getCryptoBufferSize()));
+        }
+    }
     closed = false;
     computePacketChunkSize();
     LeaseRenewer::GetLeaseRenewer().StartRenew(filesystem);
@@ -278,7 +316,12 @@ void OutputStreamImpl::append(const char * buf, int64_t size) {
 
 void OutputStreamImpl::appendInternal(const char * buf, int64_t size) {
     int64_t todo = size;
+	std::string bufEncode;
+    if (fileStatus.isFileEncrypted()) {
+		bufEncode = cryptoCodec->encode(buf, size);
+        buf = bufEncode.c_str();
 
+    }
     while (todo > 0) {
         int batch = buffer.size() - position;
         batch = batch < todo ? batch : static_cast<int>(todo);

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/OutputStreamImpl.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/OutputStreamImpl.h b/depends/libhdfs3/src/client/OutputStreamImpl.h
index 808ff80..8ffb5d1 100644
--- a/depends/libhdfs3/src/client/OutputStreamImpl.h
+++ b/depends/libhdfs3/src/client/OutputStreamImpl.h
@@ -35,6 +35,8 @@
 #include "server/LocatedBlock.h"
 #include "SessionConfig.h"
 #include "Thread.h"
+#include "CryptoCodec.h"
+#include "KmsClientProvider.h"
 #ifdef MOCK
 #include "PipelineStub.h"
 #endif
@@ -104,6 +106,26 @@ public:
      */
     void setError(const exception_ptr & error);
 
+    /**
+     * Get KmsClientProvider.
+     */
+    shared_ptr<KmsClientProvider> getKmsClientProvider();
+
+    /**
+     * Set KmsClientProvider.
+     */
+    void setKmsClientProvider(shared_ptr<KmsClientProvider> kcp);
+	
+    /**
+     * Get CryptoCodec.
+     */
+    shared_ptr<CryptoCodec> getCryptoCodec();
+	
+    /**
+     * Set CryptoCodec.
+     */
+    void setCryptoCodec(shared_ptr<CryptoCodec> cryptoCodec);
+
 private:
     void appendChunkToPacket(const char * buf, int size);
     void appendInternal(const char * buf, int64_t size);
@@ -153,6 +175,10 @@ private:
     std::vector<char> buffer;
     steady_clock::time_point lastSend;
     //thread heartBeatSender;
+    FileStatus fileStatus;
+    shared_ptr<CryptoCodec> cryptoCodec;
+    shared_ptr<KmsClientProvider> kcp;
+    shared_ptr<RpcAuth> auth;
 
     friend class Pipeline;
 #ifdef MOCK

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/client/UserInfo.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/client/UserInfo.h b/depends/libhdfs3/src/client/UserInfo.h
index 0489171..b8f506c 100644
--- a/depends/libhdfs3/src/client/UserInfo.h
+++ b/depends/libhdfs3/src/client/UserInfo.h
@@ -59,10 +59,10 @@ public:
         this->effectiveUser = KerberosName(effectiveUser);
     }
 
-	std::string getKrbName() const {
-	    return effectiveUser.getName();
-      
-	}
+    std::string getKrbName() const {
+        return effectiveUser.getName();
+
+    }
     std::string getPrincipal() const {
         return effectiveUser.getPrincipal();
     }

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/common/SessionConfig.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/common/SessionConfig.cpp b/depends/libhdfs3/src/common/SessionConfig.cpp
index d531036..9495642 100644
--- a/depends/libhdfs3/src/common/SessionConfig.cpp
+++ b/depends/libhdfs3/src/common/SessionConfig.cpp
@@ -127,17 +127,18 @@ SessionConfig::SessionConfig(const Config & conf) {
         }, {
             &socketCacheCapacity, "dfs.client.socketcache.capacity", 16, bind(CheckRangeGE<int32_t>, _1, _2, 0)
         }, {
-			&cryptoBufferSize, "hadoop.security.crypto.buffer.size", 8192
-		},{
-			&httpRequestRetryTimes, "kms.send.request.retry.times", 4
-		}
+            &cryptoBufferSize, "hadoop.security.crypto.buffer.size", 8192
+        }, {
+            &httpRequestRetryTimes, "kms.send.request.retry.times", 4
+        }
     };
     ConfigDefault<int64_t> i64Values [] = {
         {
             &defaultBlockSize, "dfs.default.blocksize", 64 * 1024 * 1024, bind(CheckMultipleOf<int64_t>, _1, _2, 512)
-        }, {
+        },
+        {
             &curlTimeout, "kms.send.request.timeout", 20L
-		}
+        }
     };
 
     ConfigDefault<std::string> strValues [] = {
@@ -146,8 +147,8 @@ SessionConfig::SessionConfig(const Config & conf) {
         {&kerberosCachePath, "hadoop.security.kerberos.ticket.cache.path", "" },
         {&logSeverity, "dfs.client.log.severity", "INFO" },
         {&domainSocketPath, "dfs.domain.socket.path", ""},
-		{&kmsUrl, "dfs.encryption.key.provider.uri", ""},
-		{&kmsAuthMethod, "hadoop.kms.authentication.type", "simple" }
+        {&kmsUrl, "dfs.encryption.key.provider.uri", "" },
+        {&kmsAuthMethod, "hadoop.kms.authentication.type", "simple" }
     };
 
     for (size_t i = 0; i < ARRAYSIZE(boolValues); ++i) {

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/src/common/SessionConfig.h
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/src/common/SessionConfig.h b/depends/libhdfs3/src/common/SessionConfig.h
index 020e99b..7722401 100644
--- a/depends/libhdfs3/src/common/SessionConfig.h
+++ b/depends/libhdfs3/src/common/SessionConfig.h
@@ -301,23 +301,23 @@ public:
       return socketCacheCapacity;
     }
 
-	const std::string& getKmsUrl() const {
-      return kmsUrl;
+    const std::string& getKmsUrl() const {
+        return kmsUrl;
     }
 
-	const std::string& getKmsMethod() const {
+    const std::string& getKmsMethod() const {
         return kmsAuthMethod;
     }
 
-	int32_t getCryptoBufferSize() const {
+    int32_t getCryptoBufferSize() const {
         return cryptoBufferSize;
     }
 
-	int32_t getHttpRequestRetryTimes() const {
+    int32_t getHttpRequestRetryTimes() const {
         return httpRequestRetryTimes;
     }
 
-	int64_t getCurlTimeOut() const {
+    int64_t getCurlTimeOut() const {
         return curlTimeout;
     }
 
@@ -379,12 +379,12 @@ public:
     int32_t packetPoolSize;
     int32_t heartBeatInterval;
     int32_t closeFileTimeout;
-	std::string kmsUrl;
-	std::string kmsAuthMethod;
-	int32_t cryptoBufferSize;
-	int32_t httpRequestRetryTimes;
-	int64_t curlTimeout;
-	
+    std::string kmsUrl;
+    std::string kmsAuthMethod;
+    int32_t cryptoBufferSize;
+    int32_t httpRequestRetryTimes;
+    int64_t curlTimeout;
+
 };
 
 }

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/test/data/function-test.xml
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/test/data/function-test.xml b/depends/libhdfs3/test/data/function-test.xml
index 4e982ab..0188af8 100644
--- a/depends/libhdfs3/test/data/function-test.xml
+++ b/depends/libhdfs3/test/data/function-test.xml
@@ -114,4 +114,19 @@
 		<name>dfs.client.socketcache.capacity</name>
 		<value>1</value>
 	</property>
+
+	<property>
+		<name>dfs.encryption.key.provider.uri</name>
+		<value>kms://http@localhost:16000/kms</value>
+	</property>
+
+	<property>
+		<name>hadoop.kms.authentication.type</name>
+		<value>simple</value>
+	</property>
+
+	<property>
+		<name>hadoop.security.crypto.buffer.size</name>
+		<value>8192</value>
+	</property>
 </configuration>

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/test/function/TestCInterface.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/test/function/TestCInterface.cpp b/depends/libhdfs3/test/function/TestCInterface.cpp
index e45aaee..56fe07e 100644
--- a/depends/libhdfs3/test/function/TestCInterface.cpp
+++ b/depends/libhdfs3/test/function/TestCInterface.cpp
@@ -21,6 +21,9 @@
  */
 #include "gtest/gtest.h"
 #include "client/hdfs.h"
+#include "client/HttpClient.h"
+#include "client/KmsClientProvider.h"
+#include "client/FileEncryptionInfo.h"
 #include "Logger.h"
 #include "SessionConfig.h"
 #include "TestUtil.h"
@@ -33,6 +36,8 @@
 #include <stdlib.h>
 #include <sstream>
 #include <iostream>
+#include <openssl/md5.h>
+#include <stdio.h>
 
 using namespace Hdfs::Internal;
 
@@ -41,7 +46,10 @@ using namespace Hdfs::Internal;
 #endif
 
 #define BASE_DIR TEST_HDFS_PREFIX"/testCInterface/"
+#define MAXDATABUFF 1024
+#define MD5LENTH 16
 
+using namespace std;
 using Hdfs::CheckBuffer;
 
 static bool ReadFully(hdfsFS fs, hdfsFile file, char * buffer, size_t length) {
@@ -92,6 +100,40 @@ static bool CreateFile(hdfsFS fs, const char * path, int64_t blockSize,
     return rc >= 0;
 }
 
+static void fileMD5(const char* strFilePath, char* result) {
+    MD5_CTX ctx;
+    int len = 0;
+    unsigned char buffer[1024] = { 0 };
+    unsigned char digest[16] = { 0 };
+    FILE *pFile = fopen(strFilePath, "rb");
+    MD5_Init(&ctx);
+    while ((len = fread(buffer, 1, 1024, pFile)) > 0) {
+        MD5_Update(&ctx, buffer, len);
+    }
+    MD5_Final(digest, &ctx);
+    fclose(pFile);
+    int i = 0;
+    char tmp[3] = { 0 };
+    for (i = 0; i < 16; i++) {
+        sprintf(tmp, "%02X", digest[i]);
+        strcat(result, tmp);
+    }
+}
+
+static void bufferMD5(const char* strFilePath, int size, char* result) {
+    unsigned char digest[16] = { 0 };
+    MD5_CTX ctx;
+    MD5_Init(&ctx);
+    MD5_Update(&ctx, strFilePath, size);
+    MD5_Final(digest, &ctx);
+    int i = 0;
+    char tmp[3] = { 0 };
+    for (i = 0; i < 16; i++) {
+        sprintf(tmp, "%02X", digest[i]);
+        strcat(result, tmp);
+    }
+}
+
 bool CheckFileContent(hdfsFS fs, const char * path, int64_t len, size_t offset) {
     hdfsFile in = hdfsOpenFile(fs, path, O_RDONLY, 0, 0, 0);
 
@@ -220,7 +262,7 @@ TEST(TestCInterfaceTDE, DISABLED_TestCreateEnRPC_Success) {
     EXPECT_TRUE(enInfo->mKeyName != NULL);
     std::cout << "----hdfsEncryptionZoneInfo----:" << " KeyName : " << enInfo->mKeyName << " Suite : " << enInfo->mSuite << " CryptoProtocolVersion : " << enInfo->mCryptoProtocolVersion << " Id : " << enInfo->mId << " Path : " << enInfo->mPath << std::endl;
     hdfsFreeEncryptionZoneInfo(enInfo, 1);
-    for (int i = 0; i <= 201; i++){
+    for (int i = 0; i <= 10; i++){
         std::stringstream newstr;
         newstr << i;
         std::string tde = "/TDE" + newstr.str();
@@ -236,11 +278,115 @@ TEST(TestCInterfaceTDE, DISABLED_TestCreateEnRPC_Success) {
     hdfsEncryptionZoneInfo * enZoneInfos = NULL;
     int num = 0;
     hdfsListEncryptionZones(fs, &num);
-    EXPECT_EQ(num, 203); 
+    EXPECT_EQ(num, 12); 
+    ASSERT_EQ(hdfsDisconnect(fs), 0);
+    hdfsFreeBuilder(bld);
+}
+
+TEST(TestCInterfaceTDE, TestAppendWithTDE_Success) {
+    hdfsFS fs = NULL;
+    hdfsEncryptionZoneInfo * enInfo = NULL;
+    char * uri = NULL;
+    setenv("LIBHDFS3_CONF", "function-test.xml", 1);
+    struct hdfsBuilder * bld = hdfsNewBuilder();
+    assert(bld != NULL);
+    hdfsBuilderSetNameNode(bld, "default");
+    fs = hdfsBuilderConnect(bld);
+    hdfsBuilderSetUserName(bld, HDFS_SUPERUSER);
+    ASSERT_TRUE(fs != NULL);
+    system("hadoop fs -rmr /TDE");
+    system("hadoop key delete keytde4append -f");
+    system("hadoop key create keytde4append");
+    system("hadoop fs -mkdir /TDE");
+    ASSERT_EQ(0, hdfsCreateEncryptionZone(fs, "/TDE", "keytde4append"));
+    enInfo = hdfsGetEZForPath(fs, "/TDE");
+    ASSERT_TRUE(enInfo != NULL);
+    EXPECT_TRUE(enInfo->mKeyName != NULL);
+    hdfsFreeEncryptionZoneInfo(enInfo, 1);
+    const char *tdefile = "/TDE/testfile";
+    ASSERT_TRUE(CreateFile(fs, tdefile, 0, 0));
+
+    const char *buffer = "hello world";
+    hdfsFile out = hdfsOpenFile(fs, tdefile, O_WRONLY | O_APPEND, 0, 0, 0);
+    ASSERT_TRUE(out != NULL)<< hdfsGetLastError();
+    EXPECT_EQ(strlen(buffer), hdfsWrite(fs, out, (const void *)buffer, strlen(buffer)))
+            << hdfsGetLastError();
+    hdfsCloseFile(fs, out);
+    FILE *file = popen("hadoop fs -cat /TDE/testfile", "r");
+    char bufGets[128];
+    while (fgets(bufGets, sizeof(bufGets), file)) {
+    }
+    pclose(file);
+    ASSERT_STREQ(bufGets, buffer);
+    system("hadoop fs -rmr /TDE");
+    system("hadoop key delete keytde4append -f");
+    ASSERT_EQ(hdfsDisconnect(fs), 0);
+    hdfsFreeBuilder(bld);
+}
+
+TEST(TestCInterfaceTDE, TestAppendWithTDELargeFiles_Success) {
+    hdfsFS fs = NULL;
+    hdfsEncryptionZoneInfo * enInfo = NULL;
+    char * uri = NULL;
+    setenv("LIBHDFS3_CONF", "function-test.xml", 1);
+    struct hdfsBuilder * bld = hdfsNewBuilder();
+    assert(bld != NULL);
+    hdfsBuilderSetNameNode(bld, "default");
+    fs = hdfsBuilderConnect(bld);
+    ASSERT_TRUE(fs != NULL);
+
+    //creake key and encryption zone
+    system("hadoop fs -rmr /TDE");
+    system("hadoop key delete keytde4append -f");
+    system("hadoop key create keytde4append");
+    system("hadoop fs -mkdir /TDE");
+    ASSERT_EQ(0, hdfsCreateEncryptionZone(fs, "/TDE", "keytde4append"));
+    enInfo = hdfsGetEZForPath(fs, "/TDE");
+    ASSERT_TRUE(enInfo != NULL);
+    EXPECT_TRUE(enInfo->mKeyName != NULL);
+    hdfsFreeEncryptionZoneInfo(enInfo, 1);
+    const char *tdefile = "/TDE/testfile";
+    ASSERT_TRUE(CreateFile(fs, tdefile, 0, 0));
+
+    int size = 1024 * 32;
+    size_t offset = 0;
+    hdfsFile out;
+    int64_t todo = size;
+    std::vector<char> buffer(size);
+    int rc = -1;
+    do {
+        if (NULL == (out = hdfsOpenFile(fs, tdefile, O_WRONLY | O_APPEND, 0, 0, 1024))) {
+            break;
+        }
+        Hdfs::FillBuffer(&buffer[0], buffer.size(), 1024);
+        buffer.push_back(0);
+        while (todo > 0) {
+            if (0 > (rc = hdfsWrite(fs, out, &buffer[offset], todo))) {
+                break;
+            }
+            todo -= rc;
+            offset += rc;
+        }
+        rc = hdfsCloseFile(fs, out);
+    } while (0);
+    system("rm -rf ./testfile");
+    system("hadoop fs -get /TDE/testfile ./");
+    char resultFile[33] = { 0 };
+    fileMD5("./testfile", resultFile);
+    std::cout << "resultFile is " << resultFile << std::endl;
+    char resultBuffer[33] = { 0 };
+    LOG(INFO, "buffer is %s", &buffer[0]);
+    bufferMD5(&buffer[0], size, resultBuffer);
+    std::cout << "result is " << resultBuffer << std::endl;
+    ASSERT_STREQ(resultFile, resultBuffer);
+    system("rm ./testfile");
+    system("hadoop fs -rmr /TDE");
+    system("hadoop key delete keytde4append -f");
     ASSERT_EQ(hdfsDisconnect(fs), 0);
     hdfsFreeBuilder(bld);
 }
 
+
 TEST(TestErrorMessage, TestErrorMessage) {
     EXPECT_NO_THROW(hdfsGetLastError());
     hdfsChown(NULL, TEST_HDFS_PREFIX, NULL, NULL);

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/test/function/TestKmsClient.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/test/function/TestKmsClient.cpp b/depends/libhdfs3/test/function/TestKmsClient.cpp
index 0f9bbc2..d997f88 100644
--- a/depends/libhdfs3/test/function/TestKmsClient.cpp
+++ b/depends/libhdfs3/test/function/TestKmsClient.cpp
@@ -45,37 +45,38 @@ using namespace Hdfs::Internal;
 
 class TestKmsClient: public ::testing::Test {
 public:
-    TestKmsClient() : conf("function-test.xml") {
+    TestKmsClient() :
+            conf("function-test.xml") {
         conf.set("hadoop.kms.authentication.type", "simple");
-		conf.set("dfs.encryption.key.provider.uri","kms://http@localhost:16000/kms");
-		sconf.reset(new SessionConfig(conf));	
-		userInfo.setRealUser("abai");
-		auth.reset(new RpcAuth(userInfo, RpcAuth::ParseMethod(sconf->getKmsMethod())));
-		hc.reset(new HttpClient());
-		kcp.reset(new KmsClientProvider(auth, sconf));
-		kcp->setHttpClient(hc);
-		fs.reset(new FileSystem(conf));
-		fs->connect();
+        conf.set("dfs.encryption.key.provider.uri",
+                "kms://http@0.0.0.0:16000/kms");
+        sconf.reset(new SessionConfig(conf));
+        userInfo.setRealUser("abai");
+        auth.reset(new RpcAuth(userInfo, RpcAuth::ParseMethod(sconf->getKmsMethod())));
+        hc.reset(new HttpClient());
+        kcp.reset(new KmsClientProvider(auth, sconf));
+        kcp->setHttpClient(hc);
+        fs.reset(new FileSystem(conf));
+        fs->connect();
     }
 
     ~TestKmsClient() {
-		try {
+        try {
             fs->disconnect();
         } catch (...) {
         }
-	}
+    }
 protected:
-    Config 							conf;
-	UserInfo			 			userInfo;
-	shared_ptr<RpcAuth> 			auth;
-	shared_ptr<HttpClient> 			hc;
-	shared_ptr<KmsClientProvider> 	kcp;
-	shared_ptr<SessionConfig>		sconf;
-	shared_ptr<FileSystem> 			fs;
+    Config conf;
+    UserInfo userInfo;
+    shared_ptr<RpcAuth> auth;
+    shared_ptr<HttpClient> hc;
+    shared_ptr<KmsClientProvider> kcp;
+    shared_ptr<SessionConfig> sconf;
+    shared_ptr<FileSystem> fs;
 };
 
-static bool CreateFile(hdfsFS fs, const char * path, int64_t blockSize,
-                       int64_t fileSize) {
+static bool CreateFile(hdfsFS fs, const char * path, int64_t blockSize, int64_t fileSize) {
     hdfsFile out;
     size_t offset = 0;
     int64_t todo = fileSize, batch;
@@ -108,99 +109,101 @@ static bool CreateFile(hdfsFS fs, const char * path, int64_t blockSize,
 
 
 TEST_F(TestKmsClient, CreateKeySuccess) {
-	std::string keyName = "testcreatekeyname";
-	std::string cipher = "AES/CTR/NoPadding";
-	int length = 128;
-	std::string material = "testCreateKey";
-	std::string description = "Test create key success.";
-	ASSERT_NO_THROW(kcp->createKey(keyName, cipher, length, material, description));
+    std::string keyName = "testcreatekeyname";
+    std::string cipher = "AES/CTR/NoPadding";
+    int length = 128;
+    std::string material = "testCreateKey";
+    std::string description = "Test create key success.";
+    ASSERT_NO_THROW(
+            kcp->createKey(keyName, cipher, length, material, description));
 }
 
 TEST_F(TestKmsClient, GetKeyMetadataSuccess) {
-	FileEncryptionInfo encryptionInfo;
+    FileEncryptionInfo encryptionInfo;
     encryptionInfo.setKeyName("testcreatekeyname");
-	ptree map = kcp->getKeyMetadata(encryptionInfo);
-	std::string keyName = map.get<std::string>("name");
-	ASSERT_STREQ("testcreatekeyname", keyName.c_str());
+    ptree map = kcp->getKeyMetadata(encryptionInfo);
+    std::string keyName = map.get < std::string > ("name");
+    ASSERT_STREQ("testcreatekeyname", keyName.c_str());
 }
 
 TEST_F(TestKmsClient, DeleteKeySuccess) {
-	FileEncryptionInfo encryptionInfo;
+    FileEncryptionInfo encryptionInfo;
     encryptionInfo.setKeyName("testcreatekeyname");
-	ASSERT_NO_THROW(kcp->deleteKey(encryptionInfo));
+    ASSERT_NO_THROW(kcp->deleteKey(encryptionInfo));
 }
 
 
 TEST_F(TestKmsClient, DecryptEncryptedKeySuccess) {
-	hdfsFS hfs = NULL;
+    hdfsFS hfs = NULL;
     struct hdfsBuilder * bld = hdfsNewBuilder();
     assert(bld != NULL);
     hdfsBuilderSetNameNode(bld, "default");
     hfs = hdfsBuilderConnect(bld);
-	
-	//create key
-	hc.reset(new HttpClient());
-	kcp.reset(new KmsClientProvider(auth, sconf));
-	kcp->setHttpClient(hc);
-	std::string keyName = "testdekeyname";
+
+    //create key
+    hc.reset(new HttpClient());
+    kcp.reset(new KmsClientProvider(auth, sconf));
+    kcp->setHttpClient(hc);
+    std::string keyName = "testdekeyname";
     std::string cipher = "AES/CTR/NoPadding";
     int length = 128;
     std::string material = "test DEK";
     std::string description = "Test DEK create key success.";
-	kcp->createKey(keyName, cipher, length, material, description);
-	
-	//delete dir
-	EXPECT_EQ(0, hdfsDelete(hfs, BASE_DIR"/testDEKey", true));
-	
-	//create dir
-	EXPECT_EQ(0, hdfsCreateDirectory(hfs, BASE_DIR"/testDEKey"));
-
-	//create encryption zone and encrypted file	
-	ASSERT_EQ(0, hdfsCreateEncryptionZone(hfs, BASE_DIR"/testDEKey", "testdekeyname"));
-	const char * tdeFile = BASE_DIR"/testDEKey/tdefile";
-	ASSERT_TRUE(CreateFile(hfs, tdeFile, 0, 0));
-
-	//decrypt encrypted key
-	hc.reset(new HttpClient());
-	kcp.reset(new KmsClientProvider(auth, sconf));
-	kcp->setHttpClient(hc);
-	FileStatus fileStatus = fs->getFileStatus(tdeFile);
-	FileEncryptionInfo *enInfo = fileStatus.getFileEncryption();
-	ptree map = kcp->decryptEncryptedKey(*enInfo);
-	std::string versionName = map.get<std::string>("versionName");
-	ASSERT_STREQ("EK", versionName.c_str());
-
-	//delete key
-	hc.reset(new HttpClient());
-	kcp.reset(new KmsClientProvider(auth, sconf));
-	kcp->setHttpClient(hc);
-	FileEncryptionInfo encryptionInfo;
+    kcp->createKey(keyName, cipher, length, material, description);
+
+    //delete dir
+    hdfsDelete(hfs, BASE_DIR"/testDEKey", true);
+
+    //create dir
+    EXPECT_EQ(0, hdfsCreateDirectory(hfs, BASE_DIR"/testDEKey"));
+
+    //create encryption zone and encrypted file
+    ASSERT_EQ(0,
+            hdfsCreateEncryptionZone(hfs, BASE_DIR"/testDEKey", "testdekeyname"));
+    const char * tdeFile = BASE_DIR"/testDEKey/tdefile";
+    ASSERT_TRUE(CreateFile(hfs, tdeFile, 0, 0));
+
+    //decrypt encrypted key
+    hc.reset(new HttpClient());
+    kcp.reset(new KmsClientProvider(auth, sconf));
+    kcp->setHttpClient(hc);
+    FileStatus fileStatus = fs->getFileStatus(tdeFile);
+    FileEncryptionInfo *enInfo = fileStatus.getFileEncryption();
+    ptree map = kcp->decryptEncryptedKey(*enInfo);
+    std::string versionName = map.get < std::string > ("versionName");
+    ASSERT_STREQ("EK", versionName.c_str());
+
+    //delete key
+    hc.reset(new HttpClient());
+    kcp.reset(new KmsClientProvider(auth, sconf));
+    kcp->setHttpClient(hc);
+    FileEncryptionInfo encryptionInfo;
     encryptionInfo.setKeyName("testdekeyname");
     kcp->deleteKey(encryptionInfo);
 
 }
 
 TEST_F(TestKmsClient, CreateKeyFailediBadUrl) {
-	std::string keyName = "testcreatekeyfailname";
+    std::string keyName = "testcreatekeyfailname";
     std::string cipher = "AES/CTR/NoPadding";
     int length = 128;
     std::string material = "testCreateKey";
-	
-	std::string url[4] = {
-		"ftp:///http@localhost:16000/kms",
-		"kms://htttp@localhost:16000/kms",
-		"kms:///httpss@localhost:16000/kms",
-		"kms:///http@localhost:16000/kms"	
-	};
-	for(int i=0; i<4; i++) {
- 		conf.set("hadoop.kms.authentication.type", "simple");
-   		conf.set("dfs.encryption.key.provider.uri", url[i]);
-		sconf.reset(new SessionConfig(conf));	
-		userInfo.setRealUser("abai");
-		auth.reset(new RpcAuth(userInfo, RpcAuth::ParseMethod(sconf->getKmsMethod())));
-		hc.reset(new HttpClient());
-		ASSERT_THROW(kcp.reset(new KmsClientProvider(auth, sconf)), HdfsIOException);
-	}
+
+    std::string url[4] = { "ftp:///http@localhost:16000/kms",
+            "kms://htttp@localhost:16000/kms",
+            "kms:///httpss@localhost:16000/kms",
+            "kms:///http@localhost:16000/kms" };
+    for (int i = 0; i < 4; i++) {
+        conf.set("hadoop.kms.authentication.type", "simple");
+        conf.set("dfs.encryption.key.provider.uri", url[i]);
+        sconf.reset(new SessionConfig(conf));
+        userInfo.setRealUser("abai");
+        auth.reset(new RpcAuth(userInfo, RpcAuth::ParseMethod(sconf->getKmsMethod())));
+        hc.reset(new HttpClient());
+        kcp.reset(new KmsClientProvider(auth, sconf));
+        ASSERT_THROW(kcp->createKey("tesTdeBadUrl", "AES/CTR/NoPadding", 128,
+                        "test DEK", "test DEK description"), HdfsIOException);
+    }
 }
 
 

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/test/function/TestOutputStream.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/test/function/TestOutputStream.cpp b/depends/libhdfs3/test/function/TestOutputStream.cpp
index e57df34..5c03354 100644
--- a/depends/libhdfs3/test/function/TestOutputStream.cpp
+++ b/depends/libhdfs3/test/function/TestOutputStream.cpp
@@ -517,7 +517,7 @@ TEST_F(TestOutputStream, TestOpenFileForWrite) {
 }
 
 
-TEST_F(TestOutputStream, DISABLE_TestOpenFileForWriteTDE){
+TEST_F(TestOutputStream, TestOpenFileForWriteTDE){
     conf.set("output.default.packetsize", 1024);
     fs = new FileSystem(conf);
     fs->connect();

http://git-wip-us.apache.org/repos/asf/incubator-hawq/blob/0d6a7440/depends/libhdfs3/test/unit/UnitTestCryptoCodec.cpp
----------------------------------------------------------------------
diff --git a/depends/libhdfs3/test/unit/UnitTestCryptoCodec.cpp b/depends/libhdfs3/test/unit/UnitTestCryptoCodec.cpp
new file mode 100644
index 0000000..36c67b1
--- /dev/null
+++ b/depends/libhdfs3/test/unit/UnitTestCryptoCodec.cpp
@@ -0,0 +1,133 @@
+/********************************************************************
+ * 2014 -
+ * open source under Apache License Version 2.0
+ ********************************************************************/
+/**
+ * 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.
+ */
+#include "gtest/gtest.h"
+#include "gmock/gmock.h"
+
+#include "client/FileSystem.h"
+#include "client/FileSystemImpl.h"
+#include "client/FileSystemInter.h"
+#include "client/OutputStream.h"
+#include "client/OutputStreamImpl.h"
+#include "client/Packet.h"
+#include "client/Pipeline.h"
+#include "DateTime.h"
+#include "MockFileSystemInter.h"
+#include "MockCryptoCodec.h"
+#include "MockKmsClientProvider.h"
+#include "MockHttpClient.h"
+#include "MockLeaseRenewer.h"
+#include "MockPipeline.h"
+#include "NamenodeStub.h"
+#include "server/ExtendedBlock.h"
+#include "TestDatanodeStub.h"
+#include "TestUtil.h"
+#include "Thread.h"
+#include "XmlConfig.h"
+#include "client/KmsClientProvider.h"
+#include <string>
+
+using namespace Hdfs;
+using namespace Hdfs::Internal;
+using namespace Hdfs::Mock;
+using namespace testing;
+using ::testing::AtLeast;
+
+
+class TestCryptoCodec: public ::testing::Test {
+public:
+    TestCryptoCodec() {
+        
+    }
+
+    ~TestCryptoCodec() {
+    }
+
+protected:
+};
+
+TEST_F(TestCryptoCodec, KmsGetKey_Success) {
+    FileEncryptionInfo encryptionInfo;
+    encryptionInfo.setKeyName("KmsName");
+    encryptionInfo.setIv("KmsIv");
+    encryptionInfo.setEzKeyVersionName("KmsVersionName");
+    encryptionInfo.setKey("KmsKey");
+    Config conf;
+    conf.set("hadoop.kms.authentication.type", "simple");
+    conf.set("dfs.encryption.key.provider.uri", "kms://http@0.0.0.0:16000/kms");
+    shared_ptr<SessionConfig> sconf(new SessionConfig(conf));
+    UserInfo userInfo;
+    userInfo.setRealUser("abai");
+    shared_ptr<RpcAuth> auth(new RpcAuth(userInfo, RpcAuth::ParseMethod(sconf->getKmsMethod())));
+
+    KmsClientProvider kcp(auth, sconf);
+    shared_ptr<MockHttpClient> hc(new MockHttpClient());
+    kcp.setHttpClient(hc);
+
+    EXPECT_CALL(*hc, post()).Times(1).WillOnce(
+            Return(hc->getPostResult(encryptionInfo)));
+
+    ptree map = kcp.decryptEncryptedKey(encryptionInfo);
+    std::string KmsKey = map.get < std::string > ("material");
+
+    ASSERT_STREQ("KmsKey", KmsKey.c_str());
+}
+
+TEST_F(TestCryptoCodec, encode_Success) {
+    FileEncryptionInfo encryptionInfo;
+    encryptionInfo.setKeyName("ESKeyName");
+    encryptionInfo.setIv("ESIv");
+    encryptionInfo.setEzKeyVersionName("ESVersionName");
+
+    Config conf;
+    conf.set("hadoop.kms.authentication.type", "simple");
+    conf.set("dfs.encryption.key.provider.uri", "kms://http@0.0.0.0:16000/kms");
+    shared_ptr<SessionConfig> sconf(new SessionConfig(conf));
+    UserInfo userInfo;
+    userInfo.setRealUser("abai");
+    shared_ptr<RpcAuth> auth(
+            new RpcAuth(userInfo, RpcAuth::ParseMethod(sconf->getKmsMethod())));
+
+    shared_ptr<MockKmsClientProvider> kcp(
+            new MockKmsClientProvider(auth, sconf));
+
+    //char buf[1024] = "encode hello world";
+    char buf[1024];
+    Hdfs::FillBuffer(buf, sizeof(buf), 2048);
+
+    int32_t bufSize = 1024;
+
+    std::string Key[2] = { "012345678901234567890123456789ab",
+            "0123456789012345"};
+    for (int i = 0; i < 2; i++) {
+        encryptionInfo.setKey(Key[i]);
+        shared_ptr<MockHttpClient> hc(new MockHttpClient());
+        kcp->setHttpClient(hc);
+        CryptoCodec es(&encryptionInfo, kcp, bufSize);
+        EXPECT_CALL(*kcp, decryptEncryptedKey(_)).Times(2).WillRepeatedly(
+                Return(kcp->getEDKResult(encryptionInfo)));
+        std::string encodeStr = es.encode(buf, strlen(buf));
+        ASSERT_NE(0, memcmp(buf, encodeStr.c_str(), strlen(buf)));
+
+        std::string decodeStr = es.decode(encodeStr.c_str(), strlen(buf));
+        ASSERT_STREQ(decodeStr.c_str(), buf);
+    }
+}