HDFS-2656. Add libwebhdfs, a pure C client based on WebHDFS. Contributed by Jaimin D Jetly and Jing Zhao

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/trunk@1382836 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
Tsz-wo Sze 2012-09-10 13:43:28 +00:00
parent 3a4dcfba57
commit 009235f4a9
25 changed files with 6176 additions and 1 deletions

View File

@ -221,6 +221,9 @@ Release 2.0.3-alpha - Unreleased
NEW FEATURES
HDFS-2656. Add libwebhdfs, a pure C client based on WebHDFS.
(Jaimin D Jetly and Jing Zhao via szetszwo)
IMPROVEMENTS
OPTIMIZATIONS

View File

@ -35,6 +35,7 @@ http://maven.apache.org/xsd/maven-4.0.0.xsd">
<kdc.resource.dir>../../hadoop-common-project/hadoop-common/src/test/resources/kdc</kdc.resource.dir>
<is.hadoop.component>true</is.hadoop.component>
<require.fuse>false</require.fuse>
<require.libwebhdfs>false</require.libwebhdfs>
</properties>
<dependencies>
@ -472,7 +473,7 @@ http://maven.apache.org/xsd/maven-4.0.0.xsd">
<mkdir dir="${project.build.directory}/native"/>
<exec executable="cmake" dir="${project.build.directory}/native"
failonerror="true">
<arg line="${basedir}/src/ -DGENERATED_JAVAH=${project.build.directory}/native/javah -DJVM_ARCH_DATA_MODEL=${sun.arch.data.model} -DREQUIRE_FUSE=${require.fuse}"/>
<arg line="${basedir}/src/ -DGENERATED_JAVAH=${project.build.directory}/native/javah -DJVM_ARCH_DATA_MODEL=${sun.arch.data.model} -DREQUIRE_LIBWEBHDFS=${require.libwebhdfs} -DREQUIRE_FUSE=${require.fuse}"/>
</exec>
<exec executable="make" dir="${project.build.directory}/native" failonerror="true">
<arg line="VERBOSE=1"/>

View File

@ -147,4 +147,7 @@ target_link_libraries(test_libhdfs_threaded
pthread
)
IF(REQUIRE_LIBWEBHDFS)
add_subdirectory(contrib/libwebhdfs)
ENDIF(REQUIRE_LIBWEBHDFS)
add_subdirectory(main/native/fuse-dfs)

View File

@ -0,0 +1,95 @@
#
# 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.
#
find_package(CURL)
if (CURL_FOUND)
include_directories(${CURL_INCLUDE_DIRS})
else (CURL_FOUND)
MESSAGE(STATUS "Failed to find CURL library.")
endif (CURL_FOUND)
set(CMAKE_MODULE_PATH ${CMAKE_MODULE_PATH}
"${CMAKE_SOURCE_DIR}/contrib/libwebhdfs/resources/")
MESSAGE("CMAKE_MODULE_PATH IS: " ${CMAKE_MODULE_PATH})
find_package(Jansson)
include_directories(${JANSSON_INCLUDE_DIR})
add_dual_library(webhdfs
src/exception.c
src/hdfs_web.c
src/hdfs_jni.c
src/jni_helper.c
src/hdfs_http_client.c
src/hdfs_http_query.c
src/hdfs_json_parser.c
)
target_link_dual_libraries(webhdfs
${JAVA_JVM_LIBRARY}
${CURL_LIBRARY}
${JANSSON_LIBRARY}
pthread
)
dual_output_directory(webhdfs target)
set(LIBWEBHDFS_VERSION "0.0.0")
set_target_properties(webhdfs PROPERTIES
SOVERSION ${LIBWEBHDFS_VERSION})
add_executable(test_libwebhdfs_ops
src/test_libwebhdfs_ops.c
)
target_link_libraries(test_libwebhdfs_ops
webhdfs
${CURL_LIBRARY}
${JAVA_JVM_LIBRARY}
${JANSSON_LIBRARY}
pthread
)
add_executable(test_libwebhdfs_read
src/test_libwebhdfs_read.c
)
target_link_libraries(test_libwebhdfs_read
webhdfs
${CURL_LIBRARY}
${JAVA_JVM_LIBRARY}
${JANSSON_LIBRARY}
pthread
)
add_executable(test_libwebhdfs_write
src/test_libwebhdfs_write.c
)
target_link_libraries(test_libwebhdfs_write
webhdfs
${CURL_LIBRARY}
${JAVA_JVM_LIBRARY}
${JANSSON_LIBRARY}
pthread
)
add_executable(test_libwebhdfs_threaded
src/test_libwebhdfs_threaded.c
)
target_link_libraries(test_libwebhdfs_threaded
webhdfs
${CURL_LIBRARY}
${JAVA_JVM_LIBRARY}
${JANSSON_LIBRARY}
pthread
)

View File

@ -0,0 +1,42 @@
# 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.
# - Try to find Jansson
# Once done this will define
# JANSSON_FOUND - System has Jansson
# JANSSON_INCLUDE_DIRS - The Jansson include directories
# JANSSON_LIBRARIES - The libraries needed to use Jansson
# JANSSON_DEFINITIONS - Compiler switches required for using Jansson
find_path(JANSSON_INCLUDE_DIR jansson.h
/usr/incluce
/usr/local/include )
find_library(JANSSON_LIBRARY NAMES jansson
PATHS /usr/lib /usr/local/lib )
set(JANSSON_LIBRARIES ${JANSSON_LIBRARY} )
set(JANSSON_INCLUDE_DIRS ${JANSSON_INCLUDE_DIR} )
include(FindPackageHandleStandardArgs)
# handle the QUIETLY and REQUIRED arguments and set JANSSON_FOUND to TRUE
# if all listed variables are TRUE
find_package_handle_standard_args(Jansson DEFAULT_MSG
JANSSON_LIBRARY JANSSON_INCLUDE_DIR)
mark_as_advanced(JANSSON_INCLUDE_DIR JANSSON_LIBRARY )

View File

@ -0,0 +1,237 @@
/**
* 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 "exception.h"
#include "webhdfs.h"
#include "jni_helper.h"
#include <inttypes.h>
#include <stdio.h>
#include <stdlib.h>
#include <string.h>
#define EXCEPTION_INFO_LEN (sizeof(gExceptionInfo)/sizeof(gExceptionInfo[0]))
struct ExceptionInfo {
const char * const name;
int noPrintFlag;
int excErrno;
};
static const struct ExceptionInfo gExceptionInfo[] = {
{
.name = "java/io/FileNotFoundException",
.noPrintFlag = NOPRINT_EXC_FILE_NOT_FOUND,
.excErrno = ENOENT,
},
{
.name = "org/apache/hadoop/security/AccessControlException",
.noPrintFlag = NOPRINT_EXC_ACCESS_CONTROL,
.excErrno = EACCES,
},
{
.name = "org/apache/hadoop/fs/UnresolvedLinkException",
.noPrintFlag = NOPRINT_EXC_UNRESOLVED_LINK,
.excErrno = ENOLINK,
},
{
.name = "org/apache/hadoop/fs/ParentNotDirectoryException",
.noPrintFlag = NOPRINT_EXC_PARENT_NOT_DIRECTORY,
.excErrno = ENOTDIR,
},
{
.name = "java/lang/IllegalArgumentException",
.noPrintFlag = NOPRINT_EXC_ILLEGAL_ARGUMENT,
.excErrno = EINVAL,
},
{
.name = "java/lang/OutOfMemoryError",
.noPrintFlag = 0,
.excErrno = ENOMEM,
},
};
int printExceptionWebV(hdfs_exception_msg *exc, int noPrintFlags, const char *fmt, va_list ap)
{
int i, noPrint, excErrno;
if (!exc) {
fprintf(stderr, "printExceptionWebV: the hdfs_exception_msg is NULL\n");
return EINTERNAL;
}
for (i = 0; i < EXCEPTION_INFO_LEN; i++) {
if (strstr(gExceptionInfo[i].name, exc->exception)) {
break;
}
}
if (i < EXCEPTION_INFO_LEN) {
noPrint = (gExceptionInfo[i].noPrintFlag & noPrintFlags);
excErrno = gExceptionInfo[i].excErrno;
} else {
noPrint = 0;
excErrno = EINTERNAL;
}
if (!noPrint) {
vfprintf(stderr, fmt, ap);
fprintf(stderr, " error:\n");
fprintf(stderr, "Exception: %s\nJavaClassName: %s\nMessage: %s\n", exc->exception, exc->javaClassName, exc->message);
}
free(exc);
return excErrno;
}
int printExceptionWeb(hdfs_exception_msg *exc, int noPrintFlags, const char *fmt, ...)
{
va_list ap;
int ret;
va_start(ap, fmt);
ret = printExceptionWebV(exc, noPrintFlags, fmt, ap);
va_end(ap);
return ret;
}
int printExceptionAndFreeV(JNIEnv *env, jthrowable exc, int noPrintFlags,
const char *fmt, va_list ap)
{
int i, noPrint, excErrno;
char *className = NULL;
jstring jStr = NULL;
jvalue jVal;
jthrowable jthr;
jthr = classNameOfObject(exc, env, &className);
if (jthr) {
fprintf(stderr, "PrintExceptionAndFree: error determining class name "
"of exception.\n");
className = strdup("(unknown)");
destroyLocalReference(env, jthr);
}
for (i = 0; i < EXCEPTION_INFO_LEN; i++) {
if (!strcmp(gExceptionInfo[i].name, className)) {
break;
}
}
if (i < EXCEPTION_INFO_LEN) {
noPrint = (gExceptionInfo[i].noPrintFlag & noPrintFlags);
excErrno = gExceptionInfo[i].excErrno;
} else {
noPrint = 0;
excErrno = EINTERNAL;
}
if (!noPrint) {
vfprintf(stderr, fmt, ap);
fprintf(stderr, " error:\n");
// We don't want to use ExceptionDescribe here, because that requires a
// pending exception. Instead, use ExceptionUtils.
jthr = invokeMethod(env, &jVal, STATIC, NULL,
"org/apache/commons/lang/exception/ExceptionUtils",
"getStackTrace", "(Ljava/lang/Throwable;)Ljava/lang/String;", exc);
if (jthr) {
fprintf(stderr, "(unable to get stack trace for %s exception: "
"ExceptionUtils::getStackTrace error.)\n", className);
destroyLocalReference(env, jthr);
} else {
jStr = jVal.l;
const char *stackTrace = (*env)->GetStringUTFChars(env, jStr, NULL);
if (!stackTrace) {
fprintf(stderr, "(unable to get stack trace for %s exception: "
"GetStringUTFChars error.)\n", className);
} else {
fprintf(stderr, "%s", stackTrace);
(*env)->ReleaseStringUTFChars(env, jStr, stackTrace);
}
}
}
destroyLocalReference(env, jStr);
destroyLocalReference(env, exc);
free(className);
return excErrno;
}
int printExceptionAndFree(JNIEnv *env, jthrowable exc, int noPrintFlags,
const char *fmt, ...)
{
va_list ap;
int ret;
va_start(ap, fmt);
ret = printExceptionAndFreeV(env, exc, noPrintFlags, fmt, ap);
va_end(ap);
return ret;
}
int printPendingExceptionAndFree(JNIEnv *env, int noPrintFlags,
const char *fmt, ...)
{
va_list ap;
int ret;
jthrowable exc;
exc = (*env)->ExceptionOccurred(env);
if (!exc) {
va_start(ap, fmt);
vfprintf(stderr, fmt, ap);
va_end(ap);
fprintf(stderr, " error: (no exception)");
ret = 0;
} else {
(*env)->ExceptionClear(env);
va_start(ap, fmt);
ret = printExceptionAndFreeV(env, exc, noPrintFlags, fmt, ap);
va_end(ap);
}
return ret;
}
jthrowable getPendingExceptionAndClear(JNIEnv *env)
{
jthrowable jthr = (*env)->ExceptionOccurred(env);
if (!jthr)
return NULL;
(*env)->ExceptionClear(env);
return jthr;
}
jthrowable newRuntimeError(JNIEnv *env, const char *fmt, ...)
{
char buf[512];
jobject out, exc;
jstring jstr;
va_list ap;
va_start(ap, fmt);
vsnprintf(buf, sizeof(buf), fmt, ap);
va_end(ap);
jstr = (*env)->NewStringUTF(env, buf);
if (!jstr) {
// We got an out of memory exception rather than a RuntimeException.
// Too bad...
return getPendingExceptionAndClear(env);
}
exc = constructNewObjectOfClass(env, &out, "RuntimeException",
"(java/lang/String;)V", jstr);
(*env)->DeleteLocalRef(env, jstr);
// Again, we'll either get an out of memory exception or the
// RuntimeException we wanted.
return (exc) ? exc : out;
}

View File

@ -0,0 +1,178 @@
/**
* 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 LIBHDFS_EXCEPTION_H
#define LIBHDFS_EXCEPTION_H
/**
* Exception handling routines for libhdfs.
*
* The convention we follow here is to clear pending exceptions as soon as they
* are raised. Never assume that the caller of your function will clean up
* after you-- do it yourself. Unhandled exceptions can lead to memory leaks
* and other undefined behavior.
*
* If you encounter an exception, return a local reference to it. The caller is
* responsible for freeing the local reference, by calling a function like
* PrintExceptionAndFree. (You can also free exceptions directly by calling
* DeleteLocalRef. However, that would not produce an error message, so it's
* usually not what you want.)
*/
#include <jni.h>
#include <stdio.h>
#include <stdlib.h>
#include <stdarg.h>
#include <search.h>
#include <pthread.h>
#include <errno.h>
/**
* Exception noprint flags
*
* Theses flags determine which exceptions should NOT be printed to stderr by
* the exception printing routines. For example, if you expect to see
* FileNotFound, you might use NOPRINT_EXC_FILE_NOT_FOUND, to avoid filling the
* logs with messages about routine events.
*
* On the other hand, if you don't expect any failures, you might pass
* PRINT_EXC_ALL.
*
* You can OR these flags together to avoid printing multiple classes of
* exceptions.
*/
#define PRINT_EXC_ALL 0x00
#define NOPRINT_EXC_FILE_NOT_FOUND 0x01
#define NOPRINT_EXC_ACCESS_CONTROL 0x02
#define NOPRINT_EXC_UNRESOLVED_LINK 0x04
#define NOPRINT_EXC_PARENT_NOT_DIRECTORY 0x08
#define NOPRINT_EXC_ILLEGAL_ARGUMENT 0x10
/**
* Exception information after calling webhdfs operations
*/
typedef struct {
const char *exception;
const char *javaClassName;
const char *message;
} hdfs_exception_msg;
/**
* Print out exception information got after calling webhdfs operations
*
* @param exc The exception information to print and free
* @param noPrintFlags Flags which determine which exceptions we should NOT
* print.
* @param fmt Printf-style format list
* @param ap Printf-style varargs
*
* @return The POSIX error number associated with the exception
* object.
*/
int printExceptionWebV(hdfs_exception_msg *exc, int noPrintFlags, const char *fmt, va_list ap);
/**
* Print out exception information got after calling webhdfs operations
*
* @param exc The exception information to print and free
* @param noPrintFlags Flags which determine which exceptions we should NOT
* print.
* @param fmt Printf-style format list
* @param ... Printf-style varargs
*
* @return The POSIX error number associated with the exception
* object.
*/
int printExceptionWeb(hdfs_exception_msg *exc, int noPrintFlags,
const char *fmt, ...) __attribute__((format(printf, 3, 4)));
/**
* Print out information about an exception and free it.
*
* @param env The JNI environment
* @param exc The exception to print and free
* @param noPrintFlags Flags which determine which exceptions we should NOT
* print.
* @param fmt Printf-style format list
* @param ap Printf-style varargs
*
* @return The POSIX error number associated with the exception
* object.
*/
int printExceptionAndFreeV(JNIEnv *env, jthrowable exc, int noPrintFlags,
const char *fmt, va_list ap);
/**
* Print out information about an exception and free it.
*
* @param env The JNI environment
* @param exc The exception to print and free
* @param noPrintFlags Flags which determine which exceptions we should NOT
* print.
* @param fmt Printf-style format list
* @param ... Printf-style varargs
*
* @return The POSIX error number associated with the exception
* object.
*/
int printExceptionAndFree(JNIEnv *env, jthrowable exc, int noPrintFlags,
const char *fmt, ...) __attribute__((format(printf, 4, 5)));
/**
* Print out information about the pending exception and free it.
*
* @param env The JNI environment
* @param noPrintFlags Flags which determine which exceptions we should NOT
* print.
* @param fmt Printf-style format list
* @param ... Printf-style varargs
*
* @return The POSIX error number associated with the exception
* object.
*/
int printPendingExceptionAndFree(JNIEnv *env, int noPrintFlags,
const char *fmt, ...) __attribute__((format(printf, 3, 4)));
/**
* Get a local reference to the pending exception and clear it.
*
* Once it is cleared, the exception will no longer be pending. The caller will
* have to decide what to do with the exception object.
*
* @param env The JNI environment
*
* @return The exception, or NULL if there was no exception
*/
jthrowable getPendingExceptionAndClear(JNIEnv *env);
/**
* Create a new runtime error.
*
* This creates (but does not throw) a new RuntimeError.
*
* @param env The JNI environment
* @param fmt Printf-style format list
* @param ... Printf-style varargs
*
* @return A local reference to a RuntimeError
*/
jthrowable newRuntimeError(JNIEnv *env, const char *fmt, ...)
__attribute__((format(printf, 2, 3)));
#endif

View File

@ -0,0 +1,101 @@
/**
* 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 LIBHDFS_NATIVE_TESTS_EXPECT_H
#define LIBHDFS_NATIVE_TESTS_EXPECT_H
#include <stdio.h>
#define EXPECT_ZERO(x) \
do { \
int __my_ret__ = x; \
if (__my_ret__) { \
int __my_errno__ = errno; \
fprintf(stderr, "TEST_ERROR: failed on line %d with return " \
"code %d (errno: %d): got nonzero from %s\n", \
__LINE__, __my_ret__, __my_errno__, #x); \
return __my_ret__; \
} \
} while (0);
#define EXPECT_NULL(x) \
do { \
void* __my_ret__ = x; \
int __my_errno__ = errno; \
if (__my_ret__ != NULL) { \
fprintf(stderr, "TEST_ERROR: failed on line %d (errno: %d): " \
"got non-NULL value %p from %s\n", \
__LINE__, __my_errno__, __my_ret__, #x); \
return -1; \
} \
} while (0);
#define EXPECT_NONNULL(x) \
do { \
void* __my_ret__ = x; \
int __my_errno__ = errno; \
if (__my_ret__ == NULL) { \
fprintf(stderr, "TEST_ERROR: failed on line %d (errno: %d): " \
"got NULL from %s\n", __LINE__, __my_errno__, #x); \
return -1; \
} \
} while (0);
#define EXPECT_NEGATIVE_ONE_WITH_ERRNO(x, e) \
do { \
int __my_ret__ = x; \
int __my_errno__ = errno; \
if (__my_ret__ != -1) { \
fprintf(stderr, "TEST_ERROR: failed on line %d with return " \
"code %d (errno: %d): expected -1 from %s\n", __LINE__, \
__my_ret__, __my_errno__, #x); \
return -1; \
} \
if (__my_errno__ != e) { \
fprintf(stderr, "TEST_ERROR: failed on line %d with return " \
"code %d (errno: %d): expected errno = %d from %s\n", \
__LINE__, __my_ret__, __my_errno__, e, #x); \
return -1; \
} \
} while (0);
#define EXPECT_NONZERO(x) \
do { \
int __my_ret__ = x; \
int __my_errno__ = errno; \
if (__my_ret__) { \
fprintf(stderr, "TEST_ERROR: failed on line %d with return " \
"code %d (errno: %d): got zero from %s\n", __LINE__, \
__my_ret__, __my_errno__, #x); \
return -1; \
} \
} while (0);
#define EXPECT_NONNEGATIVE(x) \
do { \
int __my_ret__ = x; \
int __my_errno__ = errno; \
if (__my_ret__ < 0) { \
fprintf(stderr, "TEST_ERROR: failed on line %d with return " \
"code %d (errno: %d): got negative return from %s\n", \
__LINE__, __my_ret__, __my_errno__, #x); \
return __my_ret__; \
} \
} while (0);
#endif

View File

@ -0,0 +1,352 @@
/**
* 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 <stdlib.h>
#include <string.h>
#include <curl/curl.h>
#include <pthread.h>
#include "hdfs_http_client.h"
static pthread_mutex_t curlInitMutex = PTHREAD_MUTEX_INITIALIZER;
static volatile int curlGlobalInited = 0;
ResponseBuffer initResponseBuffer() {
ResponseBuffer info = (ResponseBuffer) calloc(1, sizeof(ResponseBufferInternal));
if (!info) {
fprintf(stderr, "Cannot allocate memory for responseInfo\n");
return NULL;
}
info->remaining = 0;
info->offset = 0;
info->content = NULL;
return info;
}
void freeResponseBuffer(ResponseBuffer buffer) {
if (buffer) {
if (buffer->content) {
free(buffer->content);
}
free(buffer);
buffer = NULL;
}
}
void freeResponse(Response resp) {
if(resp) {
freeResponseBuffer(resp->body);
freeResponseBuffer(resp->header);
free(resp);
resp = NULL;
}
}
/* Callback for allocating local buffer and reading data to local buffer */
static size_t writefunc(void *ptr, size_t size, size_t nmemb, ResponseBuffer rbuffer) {
if (size * nmemb < 1) {
return 0;
}
if (!rbuffer) {
fprintf(stderr, "In writefunc, ResponseBuffer is NULL.\n");
return -1;
}
if (rbuffer->remaining < size * nmemb) {
rbuffer->content = realloc(rbuffer->content, rbuffer->offset + size * nmemb + 1);
if (rbuffer->content == NULL) {
return -1;
}
rbuffer->remaining = size * nmemb;
}
memcpy(rbuffer->content + rbuffer->offset, ptr, size * nmemb);
rbuffer->offset += size * nmemb;
(rbuffer->content)[rbuffer->offset] = '\0';
rbuffer->remaining -= size * nmemb;
return size * nmemb;
}
/**
* Callback for reading data to buffer provided by user,
* thus no need to reallocate buffer.
*/
static size_t writefunc_withbuffer(void *ptr, size_t size, size_t nmemb, ResponseBuffer rbuffer) {
if (size * nmemb < 1) {
return 0;
}
if (!rbuffer || !rbuffer->content) {
fprintf(stderr, "In writefunc_withbuffer, the buffer provided by user is NULL.\n");
return 0;
}
size_t toCopy = rbuffer->remaining < (size * nmemb) ? rbuffer->remaining : (size * nmemb);
memcpy(rbuffer->content + rbuffer->offset, ptr, toCopy);
rbuffer->offset += toCopy;
rbuffer->remaining -= toCopy;
return toCopy;
}
//callback for writing data to remote peer
static size_t readfunc(void *ptr, size_t size, size_t nmemb, void *stream) {
if (size * nmemb < 1) {
fprintf(stderr, "In readfunc callback: size * nmemb == %ld\n", size * nmemb);
return 0;
}
webhdfsBuffer *wbuffer = (webhdfsBuffer *) stream;
pthread_mutex_lock(&wbuffer->writeMutex);
while (wbuffer->remaining == 0) {
/*
* the current remainning bytes to write is 0,
* check whether need to finish the transfer
* if yes, return 0; else, wait
*/
if (wbuffer->closeFlag) {
//we can close the transfer now
fprintf(stderr, "CloseFlag is set, ready to close the transfer\n");
pthread_mutex_unlock(&wbuffer->writeMutex);
return 0;
} else {
// len == 0 indicates that user's buffer has been transferred
pthread_cond_signal(&wbuffer->transfer_finish);
pthread_cond_wait(&wbuffer->newwrite_or_close, &wbuffer->writeMutex);
}
}
if(wbuffer->remaining > 0 && !wbuffer->closeFlag) {
size_t copySize = wbuffer->remaining < size * nmemb ? wbuffer->remaining : size * nmemb;
memcpy(ptr, wbuffer->wbuffer + wbuffer->offset, copySize);
wbuffer->offset += copySize;
wbuffer->remaining -= copySize;
pthread_mutex_unlock(&wbuffer->writeMutex);
return copySize;
} else {
fprintf(stderr, "Webhdfs buffer is %ld, it should be a positive value!\n", wbuffer->remaining);
pthread_mutex_unlock(&wbuffer->writeMutex);
return 0;
}
}
static void initCurlGlobal() {
if (!curlGlobalInited) {
pthread_mutex_lock(&curlInitMutex);
if (!curlGlobalInited) {
curl_global_init(CURL_GLOBAL_ALL);
curlGlobalInited = 1;
}
pthread_mutex_unlock(&curlInitMutex);
}
}
static Response launchCmd(char *url, enum HttpHeader method, enum Redirect followloc) {
CURL *curl;
CURLcode res;
Response resp;
resp = (Response) calloc(1, sizeof(*resp));
if (!resp) {
return NULL;
}
resp->body = initResponseBuffer();
resp->header = initResponseBuffer();
initCurlGlobal();
curl = curl_easy_init(); /* get a curl handle */
if(curl) {
curl_easy_setopt(curl, CURLOPT_WRITEFUNCTION, writefunc);
curl_easy_setopt(curl, CURLOPT_WRITEDATA, resp->body);
curl_easy_setopt(curl, CURLOPT_HEADERFUNCTION, writefunc);
curl_easy_setopt(curl, CURLOPT_WRITEHEADER, resp->header);
curl_easy_setopt(curl, CURLOPT_URL, url); /* specify target URL */
switch(method) {
case GET:
break;
case PUT:
curl_easy_setopt(curl,CURLOPT_CUSTOMREQUEST,"PUT");
break;
case POST:
curl_easy_setopt(curl,CURLOPT_CUSTOMREQUEST,"POST");
break;
case DELETE:
curl_easy_setopt(curl,CURLOPT_CUSTOMREQUEST,"DELETE");
break;
default:
fprintf(stderr, "\nHTTP method not defined\n");
exit(EXIT_FAILURE);
}
if(followloc == YES) {
curl_easy_setopt(curl, CURLOPT_FOLLOWLOCATION, 1);
}
res = curl_easy_perform(curl); /* Now run the curl handler */
if(res != CURLE_OK) {
fprintf(stderr, "preform the URL %s failed\n", url);
return NULL;
}
curl_easy_cleanup(curl);
}
return resp;
}
static Response launchRead_internal(char *url, enum HttpHeader method, enum Redirect followloc, Response resp) {
if (!resp || !resp->body || !resp->body->content) {
fprintf(stderr, "The user provided buffer should not be NULL!\n");
return NULL;
}
CURL *curl;
CURLcode res;
initCurlGlobal();
curl = curl_easy_init(); /* get a curl handle */
if(curl) {
curl_easy_setopt(curl, CURLOPT_WRITEFUNCTION, writefunc_withbuffer);
curl_easy_setopt(curl, CURLOPT_WRITEDATA, resp->body);
curl_easy_setopt(curl, CURLOPT_HEADERFUNCTION, writefunc);
curl_easy_setopt(curl, CURLOPT_WRITEHEADER, resp->header);
curl_easy_setopt(curl, CURLOPT_URL, url); /* specify target URL */
if(followloc == YES) {
curl_easy_setopt(curl, CURLOPT_FOLLOWLOCATION, 1);
}
res = curl_easy_perform(curl); /* Now run the curl handler */
if(res != CURLE_OK && res != CURLE_PARTIAL_FILE) {
fprintf(stderr, "preform the URL %s failed\n", url);
return NULL;
}
curl_easy_cleanup(curl);
}
return resp;
}
static Response launchWrite(const char *url, enum HttpHeader method, webhdfsBuffer *uploadBuffer) {
if (!uploadBuffer) {
fprintf(stderr, "upload buffer is NULL!\n");
errno = EINVAL;
return NULL;
}
initCurlGlobal();
CURLcode res;
Response response = (Response) calloc(1, sizeof(*response));
if (!response) {
fprintf(stderr, "failed to allocate memory for response\n");
return NULL;
}
response->body = initResponseBuffer();
response->header = initResponseBuffer();
//connect to the datanode in order to create the lease in the namenode
CURL *curl = curl_easy_init();
if (!curl) {
fprintf(stderr, "Failed to initialize the curl handle.\n");
return NULL;
}
curl_easy_setopt(curl, CURLOPT_URL, url);
if(curl) {
curl_easy_setopt(curl, CURLOPT_WRITEFUNCTION, writefunc);
curl_easy_setopt(curl, CURLOPT_WRITEDATA, response->body);
curl_easy_setopt(curl, CURLOPT_HEADERFUNCTION, writefunc);
curl_easy_setopt(curl, CURLOPT_WRITEHEADER, response->header);
curl_easy_setopt(curl, CURLOPT_READFUNCTION, readfunc);
curl_easy_setopt(curl, CURLOPT_READDATA, uploadBuffer);
curl_easy_setopt(curl, CURLOPT_UPLOAD, 1L);
curl_easy_setopt(curl, CURLOPT_VERBOSE, 1);
struct curl_slist *chunk = NULL;
chunk = curl_slist_append(chunk, "Transfer-Encoding: chunked");
res = curl_easy_setopt(curl, CURLOPT_HTTPHEADER, chunk);
chunk = curl_slist_append(chunk, "Expect:");
res = curl_easy_setopt(curl, CURLOPT_HTTPHEADER, chunk);
switch(method) {
case GET:
break;
case PUT:
curl_easy_setopt(curl,CURLOPT_CUSTOMREQUEST,"PUT");
break;
case POST:
curl_easy_setopt(curl,CURLOPT_CUSTOMREQUEST,"POST");
break;
case DELETE:
curl_easy_setopt(curl,CURLOPT_CUSTOMREQUEST,"DELETE");
break;
default:
fprintf(stderr, "\nHTTP method not defined\n");
exit(EXIT_FAILURE);
}
res = curl_easy_perform(curl);
curl_slist_free_all(chunk);
curl_easy_cleanup(curl);
}
return response;
}
Response launchMKDIR(char *url) {
return launchCmd(url, PUT, NO);
}
Response launchRENAME(char *url) {
return launchCmd(url, PUT, NO);
}
Response launchGFS(char *url) {
return launchCmd(url, GET, NO);
}
Response launchLS(char *url) {
return launchCmd(url, GET, NO);
}
Response launchCHMOD(char *url) {
return launchCmd(url, PUT, NO);
}
Response launchCHOWN(char *url) {
return launchCmd(url, PUT, NO);
}
Response launchDELETE(char *url) {
return launchCmd(url, DELETE, NO);
}
Response launchOPEN(char *url, Response resp) {
return launchRead_internal(url, GET, YES, resp);
}
Response launchUTIMES(char *url) {
return launchCmd(url, PUT, NO);
}
Response launchNnWRITE(char *url) {
return launchCmd(url, PUT, NO);
}
Response launchNnAPPEND(char *url) {
return launchCmd(url, POST, NO);
}
Response launchDnWRITE(const char *url, webhdfsBuffer *buffer) {
return launchWrite(url, PUT, buffer);
}
Response launchDnAPPEND(const char *url, webhdfsBuffer *buffer) {
return launchWrite(url, POST, buffer);
}
Response launchSETREPLICATION(char *url) {
return launchCmd(url, PUT, NO);
}

View File

@ -0,0 +1,74 @@
/**
* 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_HTTP_CLIENT_H_
#define _HDFS_HTTP_CLIENT_H_
#include "webhdfs.h"
#include <curl/curl.h>
enum HttpHeader {
GET,
PUT,
POST,
DELETE
};
enum Redirect {
YES,
NO
};
typedef struct {
char *content;
size_t remaining;
size_t offset;
} ResponseBufferInternal;
typedef ResponseBufferInternal *ResponseBuffer;
/**
* The response got through webhdfs
*/
typedef struct {
ResponseBuffer body;
ResponseBuffer header;
}* Response;
ResponseBuffer initResponseBuffer();
void freeResponseBuffer(ResponseBuffer buffer);
void freeResponse(Response resp);
Response launchMKDIR(char *url);
Response launchRENAME(char *url);
Response launchCHMOD(char *url);
Response launchGFS(char *url);
Response launchLS(char *url);
Response launchDELETE(char *url);
Response launchCHOWN(char *url);
Response launchOPEN(char *url, Response resp);
Response launchUTIMES(char *url);
Response launchNnWRITE(char *url);
Response launchDnWRITE(const char *url, webhdfsBuffer *buffer);
Response launchNnAPPEND(char *url);
Response launchSETREPLICATION(char *url);
Response launchDnAPPEND(const char *url, webhdfsBuffer *buffer);
#endif //_HDFS_HTTP_CLIENT_H_

View File

@ -0,0 +1,254 @@
/**
* 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 "hdfs_http_query.h"
#include <math.h>
#include <stdlib.h>
#include <string.h>
#include <stdio.h>
#include <errno.h>
#define NUM_OF_PERMISSION_BITS 4
#define NUM_OF_PORT_BITS 6
#define NUM_OF_REPLICATION_BITS 6
static char *prepareQUERY(const char *host, int nnPort, const char *srcpath, const char *OP, const char *user) {
size_t length;
char *url;
const char *const protocol = "http://";
const char *const prefix = "/webhdfs/v1";
char *temp;
char *port;
port= (char*) malloc(NUM_OF_PORT_BITS);
if (!port) {
return NULL;
}
sprintf(port,"%d",nnPort);
if (user != NULL) {
length = strlen(protocol) + strlen(host) + strlen(":") + strlen(port) + strlen(prefix) + strlen(srcpath) + strlen ("?op=") + strlen(OP) + strlen("&user.name=") + strlen(user);
} else {
length = strlen(protocol) + strlen(host) + strlen(":") + strlen(port) + strlen(prefix) + strlen(srcpath) + strlen ("?op=") + strlen(OP);
}
temp = (char*) malloc(length + 1);
if (!temp) {
return NULL;
}
strcpy(temp,protocol);
temp = strcat(temp,host);
temp = strcat(temp,":");
temp = strcat(temp,port);
temp = strcat(temp,prefix);
temp = strcat(temp,srcpath);
temp = strcat(temp,"?op=");
temp = strcat(temp,OP);
if (user) {
temp = strcat(temp,"&user.name=");
temp = strcat(temp,user);
}
url = temp;
return url;
}
static int decToOctal(int decNo) {
int octNo=0;
int expo =0;
while (decNo != 0) {
octNo = ((decNo % 8) * pow(10,expo)) + octNo;
decNo = decNo / 8;
expo++;
}
return octNo;
}
char *prepareMKDIR(const char *host, int nnPort, const char *dirsubpath, const char *user) {
return prepareQUERY(host, nnPort, dirsubpath, "MKDIRS", user);
}
char *prepareMKDIRwithMode(const char *host, int nnPort, const char *dirsubpath, int mode, const char *user) {
char *url;
char *permission;
permission = (char*) malloc(NUM_OF_PERMISSION_BITS);
if (!permission) {
return NULL;
}
mode = decToOctal(mode);
sprintf(permission,"%d",mode);
url = prepareMKDIR(host, nnPort, dirsubpath, user);
url = realloc(url,(strlen(url) + strlen("&permission=") + strlen(permission) + 1));
if (!url) {
return NULL;
}
url = strcat(url,"&permission=");
url = strcat(url,permission);
return url;
}
char *prepareRENAME(const char *host, int nnPort, const char *srcpath, const char *destpath, const char *user) {
char *url;
url = prepareQUERY(host, nnPort, srcpath, "RENAME", user);
url = realloc(url,(strlen(url) + strlen("&destination=") + strlen(destpath) + 1));
if (!url) {
return NULL;
}
url = strcat(url,"&destination=");
url = strcat(url,destpath);
return url;
}
char *prepareGFS(const char *host, int nnPort, const char *dirsubpath, const char *user) {
return (prepareQUERY(host, nnPort, dirsubpath, "GETFILESTATUS", user));
}
char *prepareLS(const char *host, int nnPort, const char *dirsubpath, const char *user) {
return (prepareQUERY(host, nnPort, dirsubpath, "LISTSTATUS", user));
}
char *prepareCHMOD(const char *host, int nnPort, const char *dirsubpath, int mode, const char *user) {
char *url;
char *permission;
permission = (char*) malloc(NUM_OF_PERMISSION_BITS);
if (!permission) {
return NULL;
}
mode &= 0x3FFF;
mode = decToOctal(mode);
sprintf(permission,"%d",mode);
url = prepareQUERY(host, nnPort, dirsubpath, "SETPERMISSION", user);
url = realloc(url,(strlen(url) + strlen("&permission=") + strlen(permission) + 1));
if (!url) {
return NULL;
}
url = strcat(url,"&permission=");
url = strcat(url,permission);
return url;
}
char *prepareDELETE(const char *host, int nnPort, const char *dirsubpath, int recursive, const char *user) {
char *url = (prepareQUERY(host, nnPort, dirsubpath, "DELETE", user));
char *recursiveFlag = (char *)malloc(6);
if (!recursive) {
strcpy(recursiveFlag, "false");
} else {
strcpy(recursiveFlag, "true");
}
url = (char *) realloc(url, strlen(url) + strlen("&recursive=") + strlen(recursiveFlag) + 1);
if (!url) {
return NULL;
}
strcat(url, "&recursive=");
strcat(url, recursiveFlag);
return url;
}
char *prepareCHOWN(const char *host, int nnPort, const char *dirsubpath, const char *owner, const char *group, const char *user) {
char *url;
url = prepareQUERY(host, nnPort, dirsubpath, "SETOWNER", user);
if (!url) {
return NULL;
}
if(owner != NULL) {
url = realloc(url,(strlen(url) + strlen("&owner=") + strlen(owner) + 1));
url = strcat(url,"&owner=");
url = strcat(url,owner);
}
if (group != NULL) {
url = realloc(url,(strlen(url) + strlen("&group=") + strlen(group) + 1));
url = strcat(url,"&group=");
url = strcat(url,group);
}
return url;
}
char *prepareOPEN(const char *host, int nnPort, const char *dirsubpath, const char *user, size_t offset, size_t length) {
char *base_url = prepareQUERY(host, nnPort, dirsubpath, "OPEN", user);
char *url = (char *) malloc(strlen(base_url) + strlen("&offset=") + 15 + strlen("&length=") + 15);
if (!url) {
return NULL;
}
sprintf(url, "%s&offset=%ld&length=%ld", base_url, offset, length);
return url;
}
char *prepareUTIMES(const char *host, int nnPort, const char *dirsubpath, long unsigned mTime, long unsigned aTime, const char *user) {
char *url;
char *modTime;
char *acsTime;
modTime = (char*) malloc(12);
acsTime = (char*) malloc(12);
url = prepareQUERY(host, nnPort, dirsubpath, "SETTIMES", user);
sprintf(modTime,"%lu",mTime);
sprintf(acsTime,"%lu",aTime);
url = realloc(url,(strlen(url) + strlen("&modificationtime=") + strlen(modTime) + strlen("&accesstime=") + strlen(acsTime) + 1));
if (!url) {
return NULL;
}
url = strcat(url, "&modificationtime=");
url = strcat(url, modTime);
url = strcat(url,"&accesstime=");
url = strcat(url, acsTime);
return url;
}
char *prepareNnWRITE(const char *host, int nnPort, const char *dirsubpath, const char *user, int16_t replication, size_t blockSize) {
char *url;
url = prepareQUERY(host, nnPort, dirsubpath, "CREATE", user);
url = realloc(url, (strlen(url) + strlen("&overwrite=true") + 1));
if (!url) {
return NULL;
}
url = strcat(url, "&overwrite=true");
if (replication > 0) {
url = realloc(url, (strlen(url) + strlen("&replication=") + 6));
if (!url) {
return NULL;
}
sprintf(url, "%s&replication=%d", url, replication);
}
if (blockSize > 0) {
url = realloc(url, (strlen(url) + strlen("&blocksize=") + 16));
if (!url) {
return NULL;
}
sprintf(url, "%s&blocksize=%ld", url, blockSize);
}
return url;
}
char *prepareNnAPPEND(const char *host, int nnPort, const char *dirsubpath, const char *user) {
return (prepareQUERY(host, nnPort, dirsubpath, "APPEND", user));
}
char *prepareSETREPLICATION(const char *host, int nnPort, const char *path, int16_t replication, const char *user)
{
char *url = prepareQUERY(host, nnPort, path, "SETREPLICATION", user);
char *replicationNum = (char *) malloc(NUM_OF_REPLICATION_BITS);
sprintf(replicationNum, "%u", replication);
url = realloc(url, strlen(url) + strlen("&replication=") + strlen(replicationNum)+ 1);
if (!url) {
return NULL;
}
url = strcat(url, "&replication=");
url = strcat(url, replicationNum);
return url;
}

View File

@ -0,0 +1,41 @@
/**
* 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_HTTP_QUERY_H_
#define _HDFS_HTTP_QUERY_H_
#include <stdint.h>
#include <stdio.h>
char *prepareMKDIR(const char *host, int nnPort, const char *dirsubpath, const char *user);
char *prepareMKDIRwithMode(const char *host, int nnPort, const char *dirsubpath, int mode, const char *user);
char *prepareRENAME(const char *host, int nnPort, const char *srcpath, const char *destpath, const char *user);
char *prepareCHMOD(const char *host, int nnPort, const char *dirsubpath, int mode, const char *user);
char *prepareGFS(const char *host, int nnPort, const char *dirsubpath, const char *user);
char *prepareLS(const char *host, int nnPort, const char *dirsubpath, const char *user);
char *prepareDELETE(const char *host, int nnPort, const char *dirsubpath, int recursive, const char *user);
char *prepareCHOWN(const char *host, int nnPort, const char *dirsubpath, const char *owner, const char *group, const char *user);
char *prepareOPEN(const char *host, int nnPort, const char *dirsubpath, const char *user, size_t offset, size_t length);
char *prepareUTIMES(const char *host, int nnPort, const char *dirsubpath, long unsigned mTime, long unsigned aTime, const char *user);
char *prepareNnWRITE(const char *host, int nnPort, const char *dirsubpath, const char *user, int16_t replication, size_t blockSize);
char *prepareNnAPPEND(const char *host, int nnPort, const char *dirsubpath, const char *user);
char *prepareSETREPLICATION(const char *host, int nnPort, const char *path, int16_t replication, const char *user);
#endif //_HDFS_HTTP_QUERY_H_

View File

@ -0,0 +1,616 @@
/**
* 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 <stdio.h>
#include <string.h>
#include "webhdfs.h"
#include "jni_helper.h"
#include "exception.h"
/* Some frequently used Java paths */
#define HADOOP_CONF "org/apache/hadoop/conf/Configuration"
#define HADOOP_PATH "org/apache/hadoop/fs/Path"
#define HADOOP_LOCALFS "org/apache/hadoop/fs/LocalFileSystem"
#define HADOOP_FS "org/apache/hadoop/fs/FileSystem"
#define HADOOP_FSSTATUS "org/apache/hadoop/fs/FsStatus"
#define HADOOP_BLK_LOC "org/apache/hadoop/fs/BlockLocation"
#define HADOOP_DFS "org/apache/hadoop/hdfs/DistributedFileSystem"
#define HADOOP_ISTRM "org/apache/hadoop/fs/FSDataInputStream"
#define HADOOP_OSTRM "org/apache/hadoop/fs/FSDataOutputStream"
#define HADOOP_STAT "org/apache/hadoop/fs/FileStatus"
#define HADOOP_FSPERM "org/apache/hadoop/fs/permission/FsPermission"
#define JAVA_NET_ISA "java/net/InetSocketAddress"
#define JAVA_NET_URI "java/net/URI"
#define JAVA_STRING "java/lang/String"
#define JAVA_VOID "V"
/* Macros for constructing method signatures */
#define JPARAM(X) "L" X ";"
#define JARRPARAM(X) "[L" X ";"
#define JMETHOD1(X, R) "(" X ")" R
#define JMETHOD2(X, Y, R) "(" X Y ")" R
#define JMETHOD3(X, Y, Z, R) "(" X Y Z")" R
#define KERBEROS_TICKET_CACHE_PATH "hadoop.security.kerberos.ticket.cache.path"
/**
* Helper function to create a org.apache.hadoop.fs.Path object.
* @param env: The JNIEnv pointer.
* @param path: The file-path for which to construct org.apache.hadoop.fs.Path
* object.
* @return Returns a jobject on success and NULL on error.
*/
static jthrowable constructNewObjectOfPath(JNIEnv *env, const char *path,
jobject *out)
{
jthrowable jthr;
jstring jPathString;
jobject jPath;
//Construct a java.lang.String object
jthr = newJavaStr(env, path, &jPathString);
if (jthr)
return jthr;
//Construct the org.apache.hadoop.fs.Path object
jthr = constructNewObjectOfClass(env, &jPath, "org/apache/hadoop/fs/Path",
"(Ljava/lang/String;)V", jPathString);
destroyLocalReference(env, jPathString);
if (jthr)
return jthr;
*out = jPath;
return NULL;
}
/**
* Set a configuration value.
*
* @param env The JNI environment
* @param jConfiguration The configuration object to modify
* @param key The key to modify
* @param value The value to set the key to
*
* @return NULL on success; exception otherwise
*/
static jthrowable hadoopConfSetStr(JNIEnv *env, jobject jConfiguration,
const char *key, const char *value)
{
jthrowable jthr;
jstring jkey = NULL, jvalue = NULL;
jthr = newJavaStr(env, key, &jkey);
if (jthr)
goto done;
jthr = newJavaStr(env, value, &jvalue);
if (jthr)
goto done;
jthr = invokeMethod(env, NULL, INSTANCE, jConfiguration,
HADOOP_CONF, "set", JMETHOD2(JPARAM(JAVA_STRING),
JPARAM(JAVA_STRING), JAVA_VOID),
jkey, jvalue);
if (jthr)
goto done;
done:
destroyLocalReference(env, jkey);
destroyLocalReference(env, jvalue);
return jthr;
}
static jthrowable hadoopConfGetStr(JNIEnv *env, jobject jConfiguration,
const char *key, char **val)
{
jthrowable jthr;
jvalue jVal;
jstring jkey = NULL, jRet = NULL;
jthr = newJavaStr(env, key, &jkey);
if (jthr)
goto done;
jthr = invokeMethod(env, &jVal, INSTANCE, jConfiguration,
HADOOP_CONF, "get", JMETHOD1(JPARAM(JAVA_STRING),
JPARAM(JAVA_STRING)), jkey);
if (jthr)
goto done;
jRet = jVal.l;
jthr = newCStr(env, jRet, val);
done:
destroyLocalReference(env, jkey);
destroyLocalReference(env, jRet);
return jthr;
}
int hdfsConfGetStr(const char *key, char **val)
{
JNIEnv *env;
int ret;
jthrowable jthr;
jobject jConfiguration = NULL;
env = getJNIEnv();
if (env == NULL) {
ret = EINTERNAL;
goto done;
}
jthr = constructNewObjectOfClass(env, &jConfiguration, HADOOP_CONF, "()V");
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsConfGetStr(%s): new Configuration", key);
goto done;
}
jthr = hadoopConfGetStr(env, jConfiguration, key, val);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsConfGetStr(%s): hadoopConfGetStr", key);
goto done;
}
ret = 0;
done:
destroyLocalReference(env, jConfiguration);
if (ret)
errno = ret;
return ret;
}
void hdfsConfStrFree(char *val)
{
free(val);
}
static jthrowable hadoopConfGetInt(JNIEnv *env, jobject jConfiguration,
const char *key, int32_t *val)
{
jthrowable jthr = NULL;
jvalue jVal;
jstring jkey = NULL;
jthr = newJavaStr(env, key, &jkey);
if (jthr)
return jthr;
jthr = invokeMethod(env, &jVal, INSTANCE, jConfiguration,
HADOOP_CONF, "getInt", JMETHOD2(JPARAM(JAVA_STRING), "I", "I"),
jkey, (jint)(*val));
destroyLocalReference(env, jkey);
if (jthr)
return jthr;
*val = jVal.i;
return NULL;
}
int hdfsConfGetInt(const char *key, int32_t *val)
{
JNIEnv *env;
int ret;
jobject jConfiguration = NULL;
jthrowable jthr;
env = getJNIEnv();
if (env == NULL) {
ret = EINTERNAL;
goto done;
}
jthr = constructNewObjectOfClass(env, &jConfiguration, HADOOP_CONF, "()V");
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsConfGetInt(%s): new Configuration", key);
goto done;
}
jthr = hadoopConfGetInt(env, jConfiguration, key, val);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsConfGetInt(%s): hadoopConfGetInt", key);
goto done;
}
ret = 0;
done:
destroyLocalReference(env, jConfiguration);
if (ret)
errno = ret;
return ret;
}
/**
* Calculate the effective URI to use, given a builder configuration.
*
* If there is not already a URI scheme, we prepend 'hdfs://'.
*
* If there is not already a port specified, and a port was given to the
* builder, we suffix that port. If there is a port specified but also one in
* the URI, that is an error.
*
* @param bld The hdfs builder object
* @param uri (out param) dynamically allocated string representing the
* effective URI
*
* @return 0 on success; error code otherwise
*/
static int calcEffectiveURI(struct hdfsBuilder *bld, char ** uri)
{
const char *scheme;
char suffix[64];
const char *lastColon;
char *u;
size_t uriLen;
if (!bld->nn_jni)
return EINVAL;
scheme = (strstr(bld->nn_jni, "://")) ? "" : "hdfs://";
if (bld->port == 0) {
suffix[0] = '\0';
} else {
lastColon = rindex(bld->nn_jni, ':');
if (lastColon && (strspn(lastColon + 1, "0123456789") ==
strlen(lastColon + 1))) {
fprintf(stderr, "port %d was given, but URI '%s' already "
"contains a port!\n", bld->port, bld->nn_jni);
return EINVAL;
}
snprintf(suffix, sizeof(suffix), ":%d", bld->port);
}
uriLen = strlen(scheme) + strlen(bld->nn_jni) + strlen(suffix);
u = malloc((uriLen + 1) * (sizeof(char)));
if (!u) {
fprintf(stderr, "calcEffectiveURI: out of memory");
return ENOMEM;
}
snprintf(u, uriLen + 1, "%s%s%s", scheme, bld->nn_jni, suffix);
*uri = u;
return 0;
}
static const char *maybeNull(const char *str)
{
return str ? str : "(NULL)";
}
const char *hdfsBuilderToStr(const struct hdfsBuilder *bld,
char *buf, size_t bufLen)
{
snprintf(buf, bufLen, "forceNewInstance=%d, nn=%s, port=%d, "
"kerbTicketCachePath=%s, userName=%s, workingDir=%s\n",
bld->forceNewInstance, maybeNull(bld->nn), bld->port,
maybeNull(bld->kerbTicketCachePath),
maybeNull(bld->userName), maybeNull(bld->workingDir));
return buf;
}
/*
* The JNI version of builderConnect, return the reflection of FileSystem
*/
jobject hdfsBuilderConnect_JNI(JNIEnv *env, struct hdfsBuilder *bld)
{
jobject jConfiguration = NULL, jFS = NULL, jURI = NULL, jCachePath = NULL;
jstring jURIString = NULL, jUserString = NULL;
jvalue jVal;
jthrowable jthr = NULL;
char *cURI = 0, buf[512];
int ret;
jobject jRet = NULL;
// jConfiguration = new Configuration();
jthr = constructNewObjectOfClass(env, &jConfiguration, HADOOP_CONF, "()V");
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)", hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
//Check what type of FileSystem the caller wants...
if (bld->nn_jni == NULL) {
// Get a local filesystem.
// Also handle the scenario where nn of hdfsBuilder is set to localhost.
if (bld->forceNewInstance) {
// fs = FileSytem#newInstanceLocal(conf);
jthr = invokeMethod(env, &jVal, STATIC, NULL, HADOOP_FS,
"newInstanceLocal", JMETHOD1(JPARAM(HADOOP_CONF),
JPARAM(HADOOP_LOCALFS)), jConfiguration);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
jFS = jVal.l;
} else {
// fs = FileSytem#getLocal(conf);
jthr = invokeMethod(env, &jVal, STATIC, NULL, HADOOP_FS, "getLocal",
JMETHOD1(JPARAM(HADOOP_CONF),
JPARAM(HADOOP_LOCALFS)),
jConfiguration);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
jFS = jVal.l;
}
} else {
if (!strcmp(bld->nn_jni, "default")) {
// jURI = FileSystem.getDefaultUri(conf)
jthr = invokeMethod(env, &jVal, STATIC, NULL, HADOOP_FS,
"getDefaultUri",
"(Lorg/apache/hadoop/conf/Configuration;)Ljava/net/URI;",
jConfiguration);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
jURI = jVal.l;
} else {
// fs = FileSystem#get(URI, conf, ugi);
ret = calcEffectiveURI(bld, &cURI);
if (ret)
goto done;
jthr = newJavaStr(env, cURI, &jURIString);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
jthr = invokeMethod(env, &jVal, STATIC, NULL, JAVA_NET_URI,
"create", "(Ljava/lang/String;)Ljava/net/URI;",
jURIString);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
jURI = jVal.l;
}
if (bld->kerbTicketCachePath) {
jthr = hadoopConfSetStr(env, jConfiguration,
KERBEROS_TICKET_CACHE_PATH, bld->kerbTicketCachePath);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
}
jthr = newJavaStr(env, bld->userName, &jUserString);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
if (bld->forceNewInstance) {
jthr = invokeMethod(env, &jVal, STATIC, NULL, HADOOP_FS,
"newInstance", JMETHOD3(JPARAM(JAVA_NET_URI),
JPARAM(HADOOP_CONF), JPARAM(JAVA_STRING),
JPARAM(HADOOP_FS)),
jURI, jConfiguration, jUserString);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
jFS = jVal.l;
} else {
jthr = invokeMethod(env, &jVal, STATIC, NULL, HADOOP_FS, "get",
JMETHOD3(JPARAM(JAVA_NET_URI), JPARAM(HADOOP_CONF),
JPARAM(JAVA_STRING), JPARAM(HADOOP_FS)),
jURI, jConfiguration, jUserString);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
jFS = jVal.l;
}
}
jRet = (*env)->NewGlobalRef(env, jFS);
if (!jRet) {
ret = printPendingExceptionAndFree(env, PRINT_EXC_ALL,
"hdfsBuilderConnect_JNI(%s)",
hdfsBuilderToStr(bld, buf, sizeof(buf)));
goto done;
}
ret = 0;
done:
// Release unnecessary local references
destroyLocalReference(env, jConfiguration);
destroyLocalReference(env, jFS);
destroyLocalReference(env, jURI);
destroyLocalReference(env, jCachePath);
destroyLocalReference(env, jURIString);
destroyLocalReference(env, jUserString);
free(cURI);
if (ret) {
errno = ret;
return NULL;
}
return jRet;
}
int hdfsDisconnect_JNI(jobject jFS)
{
// JAVA EQUIVALENT:
// fs.close()
//Get the JNIEnv* corresponding to current thread
JNIEnv* env = getJNIEnv();
int ret;
if (env == NULL) {
errno = EINTERNAL;
return -1;
}
//Sanity check
if (jFS == NULL) {
errno = EBADF;
return -1;
}
jthrowable jthr = invokeMethod(env, NULL, INSTANCE, jFS, HADOOP_FS,
"close", "()V");
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsDisconnect: FileSystem#close");
} else {
ret = 0;
}
(*env)->DeleteGlobalRef(env, jFS);
if (ret) {
errno = ret;
return -1;
}
return 0;
}
static int hdfsCopyImpl(hdfsFS srcFS, const char* src, hdfsFS dstFS,
const char* dst, jboolean deleteSource)
{
//JAVA EQUIVALENT
// FileUtil#copy(srcFS, srcPath, dstFS, dstPath,
// deleteSource = false, conf)
//Get the JNIEnv* corresponding to current thread
JNIEnv* env = getJNIEnv();
if (env == NULL) {
errno = EINTERNAL;
return -1;
}
//In libwebhdfs, the hdfsFS derived from hdfsBuilderConnect series functions
//is actually a hdfsBuilder instance containing address information of NameNode.
//Thus here we need to use JNI to get the real java FileSystem objects.
jobject jSrcFS = hdfsBuilderConnect_JNI(env, (struct hdfsBuilder *) srcFS);
jobject jDstFS = hdfsBuilderConnect_JNI(env, (struct hdfsBuilder *) dstFS);
//Parameters
jobject jConfiguration = NULL, jSrcPath = NULL, jDstPath = NULL;
jthrowable jthr;
jvalue jVal;
int ret;
jthr = constructNewObjectOfPath(env, src, &jSrcPath);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsCopyImpl(src=%s): constructNewObjectOfPath", src);
goto done;
}
jthr = constructNewObjectOfPath(env, dst, &jDstPath);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsCopyImpl(dst=%s): constructNewObjectOfPath", dst);
goto done;
}
//Create the org.apache.hadoop.conf.Configuration object
jthr = constructNewObjectOfClass(env, &jConfiguration,
HADOOP_CONF, "()V");
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsCopyImpl: Configuration constructor");
goto done;
}
//FileUtil#copy
jthr = invokeMethod(env, &jVal, STATIC,
NULL, "org/apache/hadoop/fs/FileUtil", "copy",
"(Lorg/apache/hadoop/fs/FileSystem;Lorg/apache/hadoop/fs/Path;"
"Lorg/apache/hadoop/fs/FileSystem;Lorg/apache/hadoop/fs/Path;"
"ZLorg/apache/hadoop/conf/Configuration;)Z",
jSrcFS, jSrcPath, jDstFS, jDstPath, deleteSource,
jConfiguration);
if (jthr) {
ret = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsCopyImpl(src=%s, dst=%s, deleteSource=%d): "
"FileUtil#copy", src, dst, deleteSource);
goto done;
}
if (!jVal.z) {
ret = EIO;
goto done;
}
ret = 0;
done:
destroyLocalReference(env, jConfiguration);
destroyLocalReference(env, jSrcPath);
destroyLocalReference(env, jDstPath);
//Disconnect src/dst FileSystem
hdfsDisconnect_JNI(jSrcFS);
hdfsDisconnect_JNI(jDstFS);
if (ret) {
errno = ret;
return -1;
}
return 0;
}
int hdfsCopy(hdfsFS srcFS, const char* src, hdfsFS dstFS, const char* dst)
{
return hdfsCopyImpl(srcFS, src, dstFS, dst, 0);
}
int hdfsMove(hdfsFS srcFS, const char* src, hdfsFS dstFS, const char* dst)
{
return hdfsCopyImpl(srcFS, src, dstFS, dst, 1);
}
tOffset hdfsGetDefaultBlockSize(hdfsFS fs)
{
// JAVA EQUIVALENT:
// fs.getDefaultBlockSize();
//Get the JNIEnv* corresponding to current thread
JNIEnv* env = getJNIEnv();
if (env == NULL) {
errno = EINTERNAL;
return -1;
}
//In libwebhdfs, the hdfsFS derived from hdfsConnect functions
//is actually a hdfsBuilder instance containing address information of NameNode.
//Thus here we need to use JNI to get the real java FileSystem objects.
jobject jFS = hdfsBuilderConnect_JNI(env, (struct hdfsBuilder *) fs);
//FileSystem#getDefaultBlockSize()
jvalue jVal;
jthrowable jthr;
jthr = invokeMethod(env, &jVal, INSTANCE, jFS, HADOOP_FS,
"getDefaultBlockSize", "()J");
if (jthr) {
errno = printExceptionAndFree(env, jthr, PRINT_EXC_ALL,
"hdfsGetDefaultBlockSize: FileSystem#getDefaultBlockSize");
//Disconnect
hdfsDisconnect_JNI(jFS);
return -1;
}
//Disconnect
hdfsDisconnect_JNI(jFS);
return jVal.j;
}

View File

@ -0,0 +1,388 @@
/**
* 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 <stdlib.h>
#include <string.h>
#include <ctype.h>
#include <jansson.h>
#include "hdfs_json_parser.h"
#include "exception.h"
hdfsFileInfo *parseJsonGFS(json_t *jobj, hdfsFileInfo *fileStat, int *numEntries, const char *operation); //Forward Declaration
static hdfsFileInfo *json_parse_array(json_t *jobj, char *key, hdfsFileInfo *fileStat, int *numEntries, const char *operation) {
int arraylen = json_array_size(jobj); //Getting the length of the array
*numEntries = arraylen;
if (!key) {
return NULL;
}
if(arraylen > 0) {
fileStat = (hdfsFileInfo *)realloc(fileStat,sizeof(hdfsFileInfo)*arraylen);
}
json_t *jvalue;
int i;
for (i=0; i< arraylen; i++) {
jvalue = json_array_get(jobj, i); //Getting the array element at position i
if (json_is_array(jvalue)) { // array within an array - program should never come here for now
json_parse_array(jvalue, NULL, &fileStat[i], numEntries, operation);
}
else if (json_is_object(jvalue)) { // program will definitely come over here
parseJsonGFS(jvalue, &fileStat[i], numEntries, operation);
}
else {
return NULL; // program will never come over here for now
}
}
*numEntries = arraylen;
return fileStat;
}
int parseBoolean(char *response) {
json_t *root;
json_error_t error;
size_t flags = 0;
int result = 0;
const char *key;
json_t *value;
root = json_loads(response, flags, &error);
void *iter = json_object_iter(root);
while(iter) {
key = json_object_iter_key(iter);
value = json_object_iter_value(iter);
switch (json_typeof(value)) {
case JSON_TRUE:
result = 1;
break;
default:
result = 0;
break;
}
iter = json_object_iter_next(root, iter);
}
return result;
}
int parseMKDIR(char *response) {
return (parseBoolean(response));
}
int parseRENAME(char *response) {
return (parseBoolean(response));
}
int parseDELETE(char *response) {
return (parseBoolean(response));
}
hdfs_exception_msg *parseJsonException(json_t *jobj) {
const char *key;
json_t *value;
hdfs_exception_msg *exception = NULL;
exception = (hdfs_exception_msg *) calloc(1, sizeof(hdfs_exception_msg));
if (!exception) {
return NULL;
}
void *iter = json_object_iter(jobj);
while (iter) {
key = json_object_iter_key(iter);
value = json_object_iter_value(iter);
if (!strcmp(key, "exception")) {
exception->exception = json_string_value(value);
} else if (!strcmp(key, "javaClassName")) {
exception->javaClassName = json_string_value(value);
} else if (!strcmp(key, "message")) {
exception->message = json_string_value(value);
}
iter = json_object_iter_next(jobj, iter);
}
return exception;
}
hdfs_exception_msg *parseException(const char *content) {
if (!content) {
return NULL;
}
json_error_t error;
size_t flags = 0;
const char *key;
json_t *value;
json_t *jobj = json_loads(content, flags, &error);
if (!jobj) {
fprintf(stderr, "JSon parsing failed\n");
return NULL;
}
void *iter = json_object_iter(jobj);
while(iter) {
key = json_object_iter_key(iter);
value = json_object_iter_value(iter);
if (!strcmp(key, "RemoteException") && json_typeof(value) == JSON_OBJECT) {
return parseJsonException(value);
}
iter = json_object_iter_next(jobj, iter);
}
return NULL;
}
hdfsFileInfo *parseJsonGFS(json_t *jobj, hdfsFileInfo *fileStat, int *numEntries, const char *operation) {
const char *tempstr;
const char *key;
json_t *value;
void *iter = json_object_iter(jobj);
while(iter) {
key = json_object_iter_key(iter);
value = json_object_iter_value(iter);
switch (json_typeof(value)) {
case JSON_INTEGER:
if(!strcmp(key,"accessTime")) {
fileStat->mLastAccess = (time_t)(json_integer_value(value)/1000);
} else if (!strcmp(key,"blockSize")) {
fileStat->mBlockSize = (tOffset)json_integer_value(value);
} else if (!strcmp(key,"length")) {
fileStat->mSize = (tOffset)json_integer_value(value);
} else if(!strcmp(key,"modificationTime")) {
fileStat->mLastMod = (time_t)(json_integer_value(value)/1000);
} else if (!strcmp(key,"replication")) {
fileStat->mReplication = (short)json_integer_value(value);
}
break;
case JSON_STRING:
if(!strcmp(key,"group")) {
fileStat->mGroup=(char *)json_string_value(value);
} else if (!strcmp(key,"owner")) {
fileStat->mOwner=(char *)json_string_value(value);
} else if (!strcmp(key,"pathSuffix")) {
fileStat->mName=(char *)json_string_value(value);
} else if (!strcmp(key,"permission")) {
tempstr=(char *)json_string_value(value);
fileStat->mPermissions = (short)strtol(tempstr,(char **)NULL,8);
} else if (!strcmp(key,"type")) {
char *cvalue = (char *)json_string_value(value);
if (!strcmp(cvalue, "DIRECTORY")) {
fileStat->mKind = kObjectKindDirectory;
} else {
fileStat->mKind = kObjectKindFile;
}
}
break;
case JSON_OBJECT:
if(!strcmp(key,"FileStatus")) {
parseJsonGFS(value, fileStat, numEntries, operation);
} else if (!strcmp(key,"FileStatuses")) {
fileStat = parseJsonGFS(value, &fileStat[0], numEntries, operation);
} else if (!strcmp(key,"RemoteException")) {
//Besides returning NULL, we also need to print the exception information
hdfs_exception_msg *exception = parseJsonException(value);
if (exception) {
errno = printExceptionWeb(exception, PRINT_EXC_ALL, "Calling WEBHDFS (%s)", operation);
}
if(fileStat != NULL) {
free(fileStat);
fileStat = NULL;
}
}
break;
case JSON_ARRAY:
if (!strcmp(key,"FileStatus")) {
fileStat = json_parse_array(value,(char *) key,fileStat,numEntries, operation);
}
break;
default:
if(fileStat != NULL) {
free(fileStat);
fileStat = NULL;
}
}
iter = json_object_iter_next(jobj, iter);
}
return fileStat;
}
int checkHeader(char *header, const char *content, const char *operation) {
char *result = NULL;
char delims[] = ":";
char *responseCode= "200 OK";
if(header == '\0' || strncmp(header, "HTTP/", strlen("HTTP/"))) {
return 0;
}
if(!(strstr(header, responseCode)) || !(header = strstr(header, "Content-Length"))) {
hdfs_exception_msg *exc = parseException(content);
if (exc) {
errno = printExceptionWeb(exc, PRINT_EXC_ALL, "Calling WEBHDFS (%s)", operation);
}
return 0;
}
result = strtok(header, delims);
result = strtok(NULL,delims);
while (isspace(*result)) {
result++;
}
if(strcmp(result,"0")) { //Content-Length should be equal to 0
return 1;
} else {
return 0;
}
}
int parseOPEN(const char *header, const char *content) {
const char *responseCode1 = "307 TEMPORARY_REDIRECT";
const char *responseCode2 = "200 OK";
if(header == '\0' || strncmp(header,"HTTP/",strlen("HTTP/"))) {
return -1;
}
if(!(strstr(header,responseCode1) && strstr(header, responseCode2))) {
hdfs_exception_msg *exc = parseException(content);
if (exc) {
//if the exception is an IOException and it is because the offset is out of the range
//do not print out the exception
if (!strcasecmp(exc->exception, "IOException") && strstr(exc->message, "out of the range")) {
return 0;
}
errno = printExceptionWeb(exc, PRINT_EXC_ALL, "Calling WEBHDFS (OPEN)");
}
return -1;
}
return 1;
}
int parseCHMOD(char *header, const char *content) {
return checkHeader(header, content, "CHMOD");
}
int parseCHOWN(char *header, const char *content) {
return checkHeader(header, content, "CHOWN");
}
int parseUTIMES(char *header, const char *content) {
return checkHeader(header, content, "UTIMES");
}
int checkIfRedirect(const char *const headerstr, const char *content, const char *operation) {
char *responseCode = "307 TEMPORARY_REDIRECT";
char * locTag = "Location";
char * tempHeader;
if(headerstr == '\0' || strncmp(headerstr,"HTTP/", 5)) {
return 0;
}
if(!(tempHeader = strstr(headerstr,responseCode))) {
//process possible exception information
hdfs_exception_msg *exc = parseException(content);
if (exc) {
errno = printExceptionWeb(exc, PRINT_EXC_ALL, "Calling WEBHDFS (%s)", operation);
}
return 0;
}
if(!(strstr(tempHeader,locTag))) {
return 0;
}
return 1;
}
int parseNnWRITE(const char *header, const char *content) {
return checkIfRedirect(header, content, "Write(NameNode)");
}
int parseNnAPPEND(const char *header, const char *content) {
return checkIfRedirect(header, content, "Append(NameNode)");
}
char *parseDnLoc(char *content) {
char delims[] = "\r\n";
char *url = NULL;
char *DnLocation = NULL;
char *savepter;
DnLocation = strtok_r(content, delims, &savepter);
while (DnLocation && strncmp(DnLocation, "Location:", strlen("Location:"))) {
DnLocation = strtok_r(NULL, delims, &savepter);
}
if (!DnLocation) {
return NULL;
}
DnLocation = strstr(DnLocation, "http");
if (!DnLocation) {
return NULL;
}
url = malloc(strlen(DnLocation) + 1);
if (!url) {
return NULL;
}
strcpy(url, DnLocation);
return url;
}
int parseDnWRITE(const char *header, const char *content) {
char *responseCode = "201 Created";
fprintf(stderr, "\nheaderstr is: %s\n", header);
if(header == '\0' || strncmp(header,"HTTP/",strlen("HTTP/"))) {
return 0;
}
if(!(strstr(header,responseCode))) {
hdfs_exception_msg *exc = parseException(content);
if (exc) {
errno = printExceptionWeb(exc, PRINT_EXC_ALL, "Calling WEBHDFS (WRITE(DataNode))");
}
return 0;
}
return 1;
}
int parseDnAPPEND(const char *header, const char *content) {
char *responseCode = "200 OK";
if(header == '\0' || strncmp(header, "HTTP/", strlen("HTTP/"))) {
return 0;
}
if(!(strstr(header, responseCode))) {
hdfs_exception_msg *exc = parseException(content);
if (exc) {
errno = printExceptionWeb(exc, PRINT_EXC_ALL, "Calling WEBHDFS (APPEND(DataNode))");
}
return 0;
}
return 1;
}
hdfsFileInfo *parseGFS(char *str, hdfsFileInfo *fileStat, int *numEntries) {
json_error_t error;
size_t flags = 0;
json_t *jobj = json_loads(str, flags, &error);
fileStat = parseJsonGFS(jobj, fileStat, numEntries, "GETPATHSTATUS/LISTSTATUS");
return fileStat;
}
int parseSETREPLICATION(char *response) {
return (parseBoolean(response));
}

View File

@ -0,0 +1,42 @@
/**
* 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_JSON_PARSER_H_
#define _HDFS_JSON_PARSER_H_
#include "webhdfs.h"
int parseMKDIR(char *response);
int parseRENAME(char *response);
int parseDELETE (char *response);
int parseSETREPLICATION(char *response);
int parseOPEN(const char *header, const char *content);
int parseNnWRITE(const char *header, const char *content);
int parseDnWRITE(const char *header, const char *content);
int parseNnAPPEND(const char *header, const char *content);
int parseDnAPPEND(const char *header, const char *content);
char* parseDnLoc(char *content);
hdfsFileInfo *parseGFS(char *response, hdfsFileInfo *fileStat, int *numEntries);
int parseCHOWN (char *header, const char *content);
int parseCHMOD (char *header, const char *content);
int parseUTIMES(char *header, const char *content);
#endif //_FUSE_JSON_PARSER_H

File diff suppressed because it is too large Load Diff

View File

@ -0,0 +1,609 @@
/**
* 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 "config.h"
#include "exception.h"
#include "jni_helper.h"
#include <stdio.h>
#include <string.h>
static pthread_mutex_t hdfsHashMutex = PTHREAD_MUTEX_INITIALIZER;
static pthread_mutex_t jvmMutex = PTHREAD_MUTEX_INITIALIZER;
static volatile int hashTableInited = 0;
#define LOCK_HASH_TABLE() pthread_mutex_lock(&hdfsHashMutex)
#define UNLOCK_HASH_TABLE() pthread_mutex_unlock(&hdfsHashMutex)
/** The Native return types that methods could return */
#define VOID 'V'
#define JOBJECT 'L'
#define JARRAYOBJECT '['
#define JBOOLEAN 'Z'
#define JBYTE 'B'
#define JCHAR 'C'
#define JSHORT 'S'
#define JINT 'I'
#define JLONG 'J'
#define JFLOAT 'F'
#define JDOUBLE 'D'
/**
* MAX_HASH_TABLE_ELEM: The maximum no. of entries in the hashtable.
* It's set to 4096 to account for (classNames + No. of threads)
*/
#define MAX_HASH_TABLE_ELEM 4096
/** Key that allows us to retrieve thread-local storage */
static pthread_key_t gTlsKey;
/** nonzero if we succeeded in initializing gTlsKey. Protected by the jvmMutex */
static int gTlsKeyInitialized = 0;
/** Pthreads thread-local storage for each library thread. */
struct hdfsTls {
JNIEnv *env;
};
/**
* The function that is called whenever a thread with libhdfs thread local data
* is destroyed.
*
* @param v The thread-local data
*/
static void hdfsThreadDestructor(void *v)
{
struct hdfsTls *tls = v;
JavaVM *vm;
JNIEnv *env = tls->env;
jint ret;
ret = (*env)->GetJavaVM(env, &vm);
if (ret) {
fprintf(stderr, "hdfsThreadDestructor: GetJavaVM failed with "
"error %d\n", ret);
(*env)->ExceptionDescribe(env);
} else {
(*vm)->DetachCurrentThread(vm);
}
free(tls);
}
void destroyLocalReference(JNIEnv *env, jobject jObject)
{
if (jObject)
(*env)->DeleteLocalRef(env, jObject);
}
static jthrowable validateMethodType(JNIEnv *env, MethType methType)
{
if (methType != STATIC && methType != INSTANCE) {
return newRuntimeError(env, "validateMethodType(methType=%d): "
"illegal method type.\n", methType);
}
return NULL;
}
jthrowable newJavaStr(JNIEnv *env, const char *str, jstring *out)
{
jstring jstr;
if (!str) {
/* Can't pass NULL to NewStringUTF: the result would be
* implementation-defined. */
*out = NULL;
return NULL;
}
jstr = (*env)->NewStringUTF(env, str);
if (!jstr) {
/* If NewStringUTF returns NULL, an exception has been thrown,
* which we need to handle. Probaly an OOM. */
return getPendingExceptionAndClear(env);
}
*out = jstr;
return NULL;
}
jthrowable newCStr(JNIEnv *env, jstring jstr, char **out)
{
const char *tmp;
if (!jstr) {
*out = NULL;
return NULL;
}
tmp = (*env)->GetStringUTFChars(env, jstr, NULL);
if (!tmp) {
return getPendingExceptionAndClear(env);
}
*out = strdup(tmp);
(*env)->ReleaseStringUTFChars(env, jstr, tmp);
return NULL;
}
static int hashTableInit(void)
{
if (!hashTableInited) {
LOCK_HASH_TABLE();
if (!hashTableInited) {
if (hcreate(MAX_HASH_TABLE_ELEM) == 0) {
fprintf(stderr, "error creating hashtable, <%d>: %s\n",
errno, strerror(errno));
return 0;
}
hashTableInited = 1;
}
UNLOCK_HASH_TABLE();
}
return 1;
}
static int insertEntryIntoTable(const char *key, void *data)
{
ENTRY e, *ep;
if (key == NULL || data == NULL) {
return 0;
}
if (! hashTableInit()) {
return -1;
}
e.data = data;
e.key = (char*)key;
LOCK_HASH_TABLE();
ep = hsearch(e, ENTER);
UNLOCK_HASH_TABLE();
if (ep == NULL) {
fprintf(stderr, "warn adding key (%s) to hash table, <%d>: %s\n",
key, errno, strerror(errno));
}
return 0;
}
static void* searchEntryFromTable(const char *key)
{
ENTRY e,*ep;
if (key == NULL) {
return NULL;
}
hashTableInit();
e.key = (char*)key;
LOCK_HASH_TABLE();
ep = hsearch(e, FIND);
UNLOCK_HASH_TABLE();
if (ep != NULL) {
return ep->data;
}
return NULL;
}
jthrowable invokeMethod(JNIEnv *env, jvalue *retval, MethType methType,
jobject instObj, const char *className,
const char *methName, const char *methSignature, ...)
{
va_list args;
jclass cls;
jmethodID mid;
jthrowable jthr;
const char *str;
char returnType;
jthr = validateMethodType(env, methType);
if (jthr)
return jthr;
jthr = globalClassReference(className, env, &cls);
if (jthr)
return jthr;
jthr = methodIdFromClass(className, methName, methSignature,
methType, env, &mid);
if (jthr)
return jthr;
str = methSignature;
while (*str != ')') str++;
str++;
returnType = *str;
va_start(args, methSignature);
if (returnType == JOBJECT || returnType == JARRAYOBJECT) {
jobject jobj = NULL;
if (methType == STATIC) {
jobj = (*env)->CallStaticObjectMethodV(env, cls, mid, args);
}
else if (methType == INSTANCE) {
jobj = (*env)->CallObjectMethodV(env, instObj, mid, args);
}
retval->l = jobj;
}
else if (returnType == VOID) {
if (methType == STATIC) {
(*env)->CallStaticVoidMethodV(env, cls, mid, args);
}
else if (methType == INSTANCE) {
(*env)->CallVoidMethodV(env, instObj, mid, args);
}
}
else if (returnType == JBOOLEAN) {
jboolean jbool = 0;
if (methType == STATIC) {
jbool = (*env)->CallStaticBooleanMethodV(env, cls, mid, args);
}
else if (methType == INSTANCE) {
jbool = (*env)->CallBooleanMethodV(env, instObj, mid, args);
}
retval->z = jbool;
}
else if (returnType == JSHORT) {
jshort js = 0;
if (methType == STATIC) {
js = (*env)->CallStaticShortMethodV(env, cls, mid, args);
}
else if (methType == INSTANCE) {
js = (*env)->CallShortMethodV(env, instObj, mid, args);
}
retval->s = js;
}
else if (returnType == JLONG) {
jlong jl = -1;
if (methType == STATIC) {
jl = (*env)->CallStaticLongMethodV(env, cls, mid, args);
}
else if (methType == INSTANCE) {
jl = (*env)->CallLongMethodV(env, instObj, mid, args);
}
retval->j = jl;
}
else if (returnType == JINT) {
jint ji = -1;
if (methType == STATIC) {
ji = (*env)->CallStaticIntMethodV(env, cls, mid, args);
}
else if (methType == INSTANCE) {
ji = (*env)->CallIntMethodV(env, instObj, mid, args);
}
retval->i = ji;
}
va_end(args);
jthr = (*env)->ExceptionOccurred(env);
if (jthr) {
(*env)->ExceptionClear(env);
return jthr;
}
return NULL;
}
jthrowable constructNewObjectOfClass(JNIEnv *env, jobject *out, const char *className,
const char *ctorSignature, ...)
{
va_list args;
jclass cls;
jmethodID mid;
jobject jobj;
jthrowable jthr;
jthr = globalClassReference(className, env, &cls);
if (jthr)
return jthr;
jthr = methodIdFromClass(className, "<init>", ctorSignature,
INSTANCE, env, &mid);
if (jthr)
return jthr;
va_start(args, ctorSignature);
jobj = (*env)->NewObjectV(env, cls, mid, args);
va_end(args);
if (!jobj)
return getPendingExceptionAndClear(env);
*out = jobj;
return NULL;
}
jthrowable methodIdFromClass(const char *className, const char *methName,
const char *methSignature, MethType methType,
JNIEnv *env, jmethodID *out)
{
jclass cls;
jthrowable jthr;
jthr = globalClassReference(className, env, &cls);
if (jthr)
return jthr;
jmethodID mid = 0;
jthr = validateMethodType(env, methType);
if (jthr)
return jthr;
if (methType == STATIC) {
mid = (*env)->GetStaticMethodID(env, cls, methName, methSignature);
}
else if (methType == INSTANCE) {
mid = (*env)->GetMethodID(env, cls, methName, methSignature);
}
if (mid == NULL) {
fprintf(stderr, "could not find method %s from class %s with "
"signature %s\n", methName, className, methSignature);
return getPendingExceptionAndClear(env);
}
*out = mid;
return NULL;
}
jthrowable globalClassReference(const char *className, JNIEnv *env, jclass *out)
{
jclass clsLocalRef;
jclass cls = searchEntryFromTable(className);
if (cls) {
*out = cls;
return NULL;
}
clsLocalRef = (*env)->FindClass(env,className);
if (clsLocalRef == NULL) {
return getPendingExceptionAndClear(env);
}
cls = (*env)->NewGlobalRef(env, clsLocalRef);
if (cls == NULL) {
(*env)->DeleteLocalRef(env, clsLocalRef);
return getPendingExceptionAndClear(env);
}
(*env)->DeleteLocalRef(env, clsLocalRef);
insertEntryIntoTable(className, cls);
*out = cls;
return NULL;
}
jthrowable classNameOfObject(jobject jobj, JNIEnv *env, char **name)
{
jthrowable jthr;
jclass cls, clsClass = NULL;
jmethodID mid;
jstring str = NULL;
const char *cstr = NULL;
char *newstr;
cls = (*env)->GetObjectClass(env, jobj);
if (cls == NULL) {
jthr = getPendingExceptionAndClear(env);
goto done;
}
clsClass = (*env)->FindClass(env, "java/lang/Class");
if (clsClass == NULL) {
jthr = getPendingExceptionAndClear(env);
goto done;
}
mid = (*env)->GetMethodID(env, clsClass, "getName", "()Ljava/lang/String;");
if (mid == NULL) {
jthr = getPendingExceptionAndClear(env);
goto done;
}
str = (*env)->CallObjectMethod(env, cls, mid);
if (str == NULL) {
jthr = getPendingExceptionAndClear(env);
goto done;
}
cstr = (*env)->GetStringUTFChars(env, str, NULL);
if (!cstr) {
jthr = getPendingExceptionAndClear(env);
goto done;
}
newstr = strdup(cstr);
if (newstr == NULL) {
jthr = newRuntimeError(env, "classNameOfObject: out of memory");
goto done;
}
*name = newstr;
jthr = NULL;
done:
destroyLocalReference(env, cls);
destroyLocalReference(env, clsClass);
if (str) {
if (cstr)
(*env)->ReleaseStringUTFChars(env, str, cstr);
(*env)->DeleteLocalRef(env, str);
}
return jthr;
}
/**
* Get the global JNI environemnt.
*
* We only have to create the JVM once. After that, we can use it in
* every thread. You must be holding the jvmMutex when you call this
* function.
*
* @return The JNIEnv on success; error code otherwise
*/
static JNIEnv* getGlobalJNIEnv(void)
{
const jsize vmBufLength = 1;
JavaVM* vmBuf[vmBufLength];
JNIEnv *env;
jint rv = 0;
jint noVMs = 0;
jthrowable jthr;
rv = JNI_GetCreatedJavaVMs(&(vmBuf[0]), vmBufLength, &noVMs);
if (rv != 0) {
fprintf(stderr, "JNI_GetCreatedJavaVMs failed with error: %d\n", rv);
return NULL;
}
if (noVMs == 0) {
//Get the environment variables for initializing the JVM
char *hadoopClassPath = getenv("CLASSPATH");
if (hadoopClassPath == NULL) {
fprintf(stderr, "Environment variable CLASSPATH not set!\n");
return NULL;
}
char *hadoopClassPathVMArg = "-Djava.class.path=";
size_t optHadoopClassPathLen = strlen(hadoopClassPath) +
strlen(hadoopClassPathVMArg) + 1;
char *optHadoopClassPath = malloc(sizeof(char)*optHadoopClassPathLen);
snprintf(optHadoopClassPath, optHadoopClassPathLen,
"%s%s", hadoopClassPathVMArg, hadoopClassPath);
// Determine the # of LIBHDFS_OPTS args
int noArgs = 1;
char *hadoopJvmArgs = getenv("LIBHDFS_OPTS");
char jvmArgDelims[] = " ";
char *str, *token, *savePtr;
if (hadoopJvmArgs != NULL) {
hadoopJvmArgs = strdup(hadoopJvmArgs);
for (noArgs = 1, str = hadoopJvmArgs; ; noArgs++, str = NULL) {
token = strtok_r(str, jvmArgDelims, &savePtr);
if (NULL == token) {
break;
}
}
free(hadoopJvmArgs);
}
// Now that we know the # args, populate the options array
JavaVMOption options[noArgs];
options[0].optionString = optHadoopClassPath;
hadoopJvmArgs = getenv("LIBHDFS_OPTS");
if (hadoopJvmArgs != NULL) {
hadoopJvmArgs = strdup(hadoopJvmArgs);
for (noArgs = 1, str = hadoopJvmArgs; ; noArgs++, str = NULL) {
token = strtok_r(str, jvmArgDelims, &savePtr);
if (NULL == token) {
break;
}
options[noArgs].optionString = token;
}
}
//Create the VM
JavaVMInitArgs vm_args;
JavaVM *vm;
vm_args.version = JNI_VERSION_1_2;
vm_args.options = options;
vm_args.nOptions = noArgs;
vm_args.ignoreUnrecognized = 1;
rv = JNI_CreateJavaVM(&vm, (void*)&env, &vm_args);
if (hadoopJvmArgs != NULL) {
free(hadoopJvmArgs);
}
free(optHadoopClassPath);
if (rv != 0) {
fprintf(stderr, "Call to JNI_CreateJavaVM failed "
"with error: %d\n", rv);
return NULL;
}
jthr = invokeMethod(env, NULL, STATIC, NULL,
"org/apache/hadoop/fs/FileSystem",
"loadFileSystems", "()V");
if (jthr) {
printExceptionAndFree(env, jthr, PRINT_EXC_ALL, "loadFileSystems");
}
}
else {
//Attach this thread to the VM
JavaVM* vm = vmBuf[0];
rv = (*vm)->AttachCurrentThread(vm, (void*)&env, 0);
if (rv != 0) {
fprintf(stderr, "Call to AttachCurrentThread "
"failed with error: %d\n", rv);
return NULL;
}
}
return env;
}
/**
* getJNIEnv: A helper function to get the JNIEnv* for the given thread.
* If no JVM exists, then one will be created. JVM command line arguments
* are obtained from the LIBHDFS_OPTS environment variable.
*
* Implementation note: we rely on POSIX thread-local storage (tls).
* This allows us to associate a destructor function with each thread, that
* will detach the thread from the Java VM when the thread terminates. If we
* failt to do this, it will cause a memory leak.
*
* However, POSIX TLS is not the most efficient way to do things. It requires a
* key to be initialized before it can be used. Since we don't know if this key
* is initialized at the start of this function, we have to lock a mutex first
* and check. Luckily, most operating systems support the more efficient
* __thread construct, which is initialized by the linker.
*
* @param: None.
* @return The JNIEnv* corresponding to the thread.
*/
JNIEnv* getJNIEnv(void)
{
JNIEnv *env;
struct hdfsTls *tls;
int ret;
#ifdef HAVE_BETTER_TLS
static __thread struct hdfsTls *quickTls = NULL;
if (quickTls)
return quickTls->env;
#endif
pthread_mutex_lock(&jvmMutex);
if (!gTlsKeyInitialized) {
ret = pthread_key_create(&gTlsKey, hdfsThreadDestructor);
if (ret) {
pthread_mutex_unlock(&jvmMutex);
fprintf(stderr, "getJNIEnv: pthread_key_create failed with "
"error %d\n", ret);
return NULL;
}
gTlsKeyInitialized = 1;
}
tls = pthread_getspecific(gTlsKey);
if (tls) {
pthread_mutex_unlock(&jvmMutex);
return tls->env;
}
env = getGlobalJNIEnv();
pthread_mutex_unlock(&jvmMutex);
if (!env) {
fprintf(stderr, "getJNIEnv: getGlobalJNIEnv failed\n");
return NULL;
}
tls = calloc(1, sizeof(struct hdfsTls));
if (!tls) {
fprintf(stderr, "getJNIEnv: OOM allocating %zd bytes\n",
sizeof(struct hdfsTls));
return NULL;
}
tls->env = env;
ret = pthread_setspecific(gTlsKey, tls);
if (ret) {
fprintf(stderr, "getJNIEnv: pthread_setspecific failed with "
"error code %d\n", ret);
hdfsThreadDestructor(tls);
return NULL;
}
#ifdef HAVE_BETTER_TLS
quickTls = tls;
#endif
return env;
}

View File

@ -0,0 +1,122 @@
/**
* 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 LIBHDFS_JNI_HELPER_H
#define LIBHDFS_JNI_HELPER_H
#include <jni.h>
#include <stdio.h>
#include <stdlib.h>
#include <stdarg.h>
#include <search.h>
#include <pthread.h>
#include <errno.h>
#define PATH_SEPARATOR ':'
/** Denote the method we want to invoke as STATIC or INSTANCE */
typedef enum {
STATIC,
INSTANCE
} MethType;
/**
* Create a new malloc'ed C string from a Java string.
*
* @param env The JNI environment
* @param jstr The Java string
* @param out (out param) the malloc'ed C string
*
* @return NULL on success; the exception otherwise
*/
jthrowable newCStr(JNIEnv *env, jstring jstr, char **out);
/**
* Create a new Java string from a C string.
*
* @param env The JNI environment
* @param str The C string
* @param out (out param) the java string
*
* @return NULL on success; the exception otherwise
*/
jthrowable newJavaStr(JNIEnv *env, const char *str, jstring *out);
/**
* Helper function to destroy a local reference of java.lang.Object
* @param env: The JNIEnv pointer.
* @param jFile: The local reference of java.lang.Object object
* @return None.
*/
void destroyLocalReference(JNIEnv *env, jobject jObject);
/** invokeMethod: Invoke a Static or Instance method.
* className: Name of the class where the method can be found
* methName: Name of the method
* methSignature: the signature of the method "(arg-types)ret-type"
* methType: The type of the method (STATIC or INSTANCE)
* instObj: Required if the methType is INSTANCE. The object to invoke
the method on.
* env: The JNIEnv pointer
* retval: The pointer to a union type which will contain the result of the
method invocation, e.g. if the method returns an Object, retval will be
set to that, if the method returns boolean, retval will be set to the
value (JNI_TRUE or JNI_FALSE), etc.
* exc: If the methods throws any exception, this will contain the reference
* Arguments (the method arguments) must be passed after methSignature
* RETURNS: -1 on error and 0 on success. If -1 is returned, exc will have
a valid exception reference, and the result stored at retval is undefined.
*/
jthrowable invokeMethod(JNIEnv *env, jvalue *retval, MethType methType,
jobject instObj, const char *className, const char *methName,
const char *methSignature, ...);
jthrowable constructNewObjectOfClass(JNIEnv *env, jobject *out, const char *className,
const char *ctorSignature, ...);
jthrowable methodIdFromClass(const char *className, const char *methName,
const char *methSignature, MethType methType,
JNIEnv *env, jmethodID *out);
jthrowable globalClassReference(const char *className, JNIEnv *env, jclass *out);
/** classNameOfObject: Get an object's class name.
* @param jobj: The object.
* @param env: The JNIEnv pointer.
* @param name: (out param) On success, will contain a string containing the
* class name. This string must be freed by the caller.
* @return NULL on success, or the exception
*/
jthrowable classNameOfObject(jobject jobj, JNIEnv *env, char **name);
/** getJNIEnv: A helper function to get the JNIEnv* for the given thread.
* If no JVM exists, then one will be created. JVM command line arguments
* are obtained from the LIBHDFS_OPTS environment variable.
* @param: None.
* @return The JNIEnv* corresponding to the thread.
* */
JNIEnv* getJNIEnv(void);
#endif /*LIBHDFS_JNI_HELPER_H*/
/**
* vim: ts=4: sw=4: et:
*/

View File

@ -0,0 +1,180 @@
/**
* 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 "expect.h"
#include "webhdfs.h"
#include <errno.h>
#include <semaphore.h>
#include <pthread.h>
#include <stdio.h>
#include <stdlib.h>
#include <string.h>
#include "hdfs_http_client.h"
#include "hdfs_http_query.h"
#include "hdfs_json_parser.h"
#include <unistd.h>
#include <curl/curl.h>
#define TLH_MAX_THREADS 100
static sem_t *tlhSem;
static const char *nn;
static const char *user;
static int port;
static const char *fileName = "/tmp/tlhData";
struct tlhThreadInfo {
/** Thread index */
int threadIdx;
/** 0 = thread was successful; error code otherwise */
int success;
/** pthread identifier */
pthread_t thread;
};
static int hdfsSingleNameNodeConnect(const char *nn, int port, const char *user, hdfsFS *fs)
{
hdfsFS hdfs;
if (port < 0) {
fprintf(stderr, "hdfsSingleNameNodeConnect: nmdGetNameNodePort "
"returned error %d\n", port);
return port;
}
hdfs = hdfsConnectAsUserNewInstance(nn, port, user);
if (!hdfs) {
return -errno;
}
*fs = hdfs;
return 0;
}
static int doTestHdfsOperations(struct tlhThreadInfo *ti, hdfsFS fs)
{
hdfsFile file;
int ret = 0;
char buffer[1024 * (ti->threadIdx + 1)];
memset(buffer, 'a', sizeof(buffer));
file = hdfsOpenFile(fs, "/tmp/thread_test.txt", O_WRONLY, 0, 0, 0);
sleep(1);
hdfsCloseFile(fs, file);
return ret;
}
static void *testHdfsOperations(void *v)
{
struct tlhThreadInfo *ti = (struct tlhThreadInfo*)v;
hdfsFS fs = NULL;
int ret;
fprintf(stderr, "testHdfsOperations(threadIdx=%d): starting\n",
ti->threadIdx);
ret = hdfsSingleNameNodeConnect(nn, port, user, &fs);
if (ret) {
fprintf(stderr, "testHdfsOperations(threadIdx=%d): "
"hdfsSingleNameNodeConnect failed with error %d.\n",
ti->threadIdx, ret);
ti->success = EIO;
return NULL;
}
ti->success = doTestHdfsOperations(ti, fs);
if (hdfsDisconnect(fs)) {
ret = errno;
fprintf(stderr, "hdfsDisconnect error %d\n", ret);
ti->success = ret;
}
return NULL;
}
static int checkFailures(struct tlhThreadInfo *ti, int tlhNumThreads)
{
int i, threadsFailed = 0;
const char *sep = "";
for (i = 0; i < tlhNumThreads; i++) {
if (ti[i].success != 0) {
threadsFailed = 1;
}
}
if (!threadsFailed) {
fprintf(stderr, "testLibHdfs: all threads succeeded. SUCCESS.\n");
return EXIT_SUCCESS;
}
fprintf(stderr, "testLibHdfs: some threads failed: [");
for (i = 0; i < tlhNumThreads; i++) {
if (ti[i].success != 0) {
fprintf(stderr, "%s%d", sep, i);
sep = ", ";
}
}
fprintf(stderr, "]. FAILURE.\n");
return EXIT_FAILURE;
}
/**
* Test that we can write a file with libhdfs and then read it back
*/
int main(int argc, const char *args[])
{
if (argc != 4) {
fprintf(stderr, "usage: test_libhdfs_threaded <namenode> <port> <username>");
return -1;
}
nn = args[1];
port = atoi(args[2]);
user = args[3];
int i, tlhNumThreads;
const char *tlhNumThreadsStr;
struct tlhThreadInfo ti[TLH_MAX_THREADS];
tlhNumThreadsStr = getenv("TLH_NUM_THREADS");
if (!tlhNumThreadsStr) {
tlhNumThreadsStr = "3";
}
tlhNumThreads = atoi(tlhNumThreadsStr);
if ((tlhNumThreads <= 0) || (tlhNumThreads > TLH_MAX_THREADS)) {
fprintf(stderr, "testLibHdfs: must have a number of threads "
"between 1 and %d inclusive, not %d\n",
TLH_MAX_THREADS, tlhNumThreads);
return EXIT_FAILURE;
}
memset(&ti[0], 0, sizeof(ti));
for (i = 0; i < tlhNumThreads; i++) {
ti[i].threadIdx = i;
}
tlhSem = sem_open("sem", O_CREAT, 0644, tlhNumThreads);
for (i = 0; i < tlhNumThreads; i++) {
fprintf(stderr, "\ncreating thread %d\n", i);
EXPECT_ZERO(pthread_create(&ti[i].thread, NULL,
testHdfsOperations, &ti[i]));
}
for (i = 0; i < tlhNumThreads; i++) {
EXPECT_ZERO(pthread_join(ti[i].thread, NULL));
}
EXPECT_ZERO(sem_close(tlhSem));
return checkFailures(ti, tlhNumThreads);
}

View File

@ -0,0 +1,504 @@
/**
* 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 "webhdfs.h"
#include <inttypes.h>
#include <jni.h>
#include <stdio.h>
#include <stdlib.h>
#include <string.h>
#include <time.h>
#include <unistd.h>
void permission_disp(short permissions, char *rtr) {
rtr[9] = '\0';
int i;
for(i=2;i>=0;i--)
{
short permissionsId = permissions >> (i * 3) & (short)7;
char* perm;
switch(permissionsId) {
case 7:
perm = "rwx"; break;
case 6:
perm = "rw-"; break;
case 5:
perm = "r-x"; break;
case 4:
perm = "r--"; break;
case 3:
perm = "-wx"; break;
case 2:
perm = "-w-"; break;
case 1:
perm = "--x"; break;
case 0:
perm = "---"; break;
default:
perm = "???";
}
strncpy(rtr, perm, 3);
rtr+=3;
}
}
int main(int argc, char **argv) {
if (argc != 2) {
fprintf(stderr, "usage: test_libwebhdfs_ops <username>\n");
return -1;
}
char buffer[32];
tSize num_written_bytes;
hdfsFS fs = hdfsConnectAsUserNewInstance("default", 50070, argv[1]);
if(!fs) {
fprintf(stderr, "Oops! Failed to connect to hdfs!\n");
exit(-1);
}
const char* writePath = "/tmp/testfile.txt";
const char* fileContents = "Hello, World!";
{
//Write tests
hdfsFile writeFile = hdfsOpenFile(fs, writePath, O_WRONLY|O_CREAT, 0, 0, 0);
if(!writeFile) {
fprintf(stderr, "Failed to open %s for writing!\n", writePath);
exit(-1);
}
fprintf(stderr, "Opened %s for writing successfully...\n", writePath);
num_written_bytes = hdfsWrite(fs, writeFile, (void*)fileContents, strlen(fileContents) + 1);
if (num_written_bytes != strlen(fileContents) + 1) {
fprintf(stderr, "Failed to write correct number of bytes - expected %d, got %d\n",
(int)(strlen(fileContents) + 1), (int)num_written_bytes);
exit(-1);
}
fprintf(stderr, "Wrote %d bytes\n", num_written_bytes);
tOffset currentPos = -1;
if ((currentPos = hdfsTell(fs, writeFile)) == -1) {
fprintf(stderr,
"Failed to get current file position correctly! Got %lld!\n",
currentPos);
exit(-1);
}
fprintf(stderr, "Current position: %lld\n", currentPos);
if (hdfsFlush(fs, writeFile)) {
fprintf(stderr, "Failed to 'flush' %s\n", writePath);
exit(-1);
}
fprintf(stderr, "Flushed %s successfully!\n", writePath);
if (hdfsHFlush(fs, writeFile)) {
fprintf(stderr, "Failed to 'hflush' %s\n", writePath);
exit(-1);
}
fprintf(stderr, "HFlushed %s successfully!\n", writePath);
hdfsCloseFile(fs, writeFile);
}
{
//Read tests
sleep(1);
const char* readPath = "/tmp/testfile.txt";
int exists = hdfsExists(fs, readPath);
if (exists) {
fprintf(stderr, "Failed to validate existence of %s\n", readPath);
exists = hdfsExists(fs, readPath);
if (exists) {
fprintf(stderr, "Still failed to validate existence of %s\n", readPath);
exit(-1);
}
}
hdfsFile readFile = hdfsOpenFile(fs, readPath, O_RDONLY, 0, 0, 0);
if (!readFile) {
fprintf(stderr, "Failed to open %s for reading!\n", readPath);
exit(-1);
}
if (!hdfsFileIsOpenForRead(readFile)) {
fprintf(stderr, "hdfsFileIsOpenForRead: we just opened a file "
"with O_RDONLY, and it did not show up as 'open for "
"read'\n");
exit(-1);
}
fprintf(stderr, "hdfsAvailable: %d\n", hdfsAvailable(fs, readFile));
tOffset seekPos = 1;
if(hdfsSeek(fs, readFile, seekPos)) {
fprintf(stderr, "Failed to seek %s for reading!\n", readPath);
exit(-1);
}
tOffset currentPos = -1;
if((currentPos = hdfsTell(fs, readFile)) != seekPos) {
fprintf(stderr,
"Failed to get current file position correctly! Got %lld!\n",
currentPos);
exit(-1);
}
fprintf(stderr, "Current position: %lld\n", currentPos);
if (!hdfsFileUsesDirectRead(readFile)) {
fprintf(stderr, "Direct read support incorrectly not detected "
"for HDFS filesystem\n");
exit(-1);
}
fprintf(stderr, "Direct read support detected for HDFS\n");
// Test the direct read path
if(hdfsSeek(fs, readFile, 0)) {
fprintf(stderr, "Failed to seek %s for reading!\n", readPath);
exit(-1);
}
memset(buffer, 0, sizeof(buffer));
tSize num_read_bytes = hdfsRead(fs, readFile, (void*)buffer,
sizeof(buffer));
if (strncmp(fileContents, buffer, strlen(fileContents)) != 0) {
fprintf(stderr, "Failed to read (direct). Expected %s but got %s (%d bytes)\n",
fileContents, buffer, num_read_bytes);
exit(-1);
}
fprintf(stderr, "Read following %d bytes:\n%s\n",
num_read_bytes, buffer);
if (hdfsSeek(fs, readFile, 0L)) {
fprintf(stderr, "Failed to seek to file start!\n");
exit(-1);
}
// Disable the direct read path so that we really go through the slow
// read path
hdfsFileDisableDirectRead(readFile);
num_read_bytes = hdfsRead(fs, readFile, (void*)buffer,
sizeof(buffer));
fprintf(stderr, "Read following %d bytes:\n%s\n",
num_read_bytes, buffer);
memset(buffer, 0, strlen(fileContents + 1));
num_read_bytes = hdfsPread(fs, readFile, 0, (void*)buffer,
sizeof(buffer));
fprintf(stderr, "Read following %d bytes:\n%s\n",
num_read_bytes, buffer);
hdfsCloseFile(fs, readFile);
}
int totalResult = 0;
int result = 0;
{
//Generic file-system operations
const char* srcPath = "/tmp/testfile.txt";
const char* dstPath = "/tmp/testfile2.txt";
const char* copyPath = "/tmp/testfile_copy.txt";
const char* movePath = "/tmp/testfile_move.txt";
fprintf(stderr, "hdfsCopy: %s\n", ((result = hdfsCopy(fs, srcPath, fs, copyPath)) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr, "hdfsMove: %s\n", ((result = hdfsMove(fs, copyPath, fs, movePath)) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr, "hdfsGetDefaultBlockSize: %lld\n", hdfsGetDefaultBlockSize(fs));
fprintf(stderr, "hdfsRename: %s\n", ((result = hdfsRename(fs, srcPath, dstPath)) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr, "hdfsRename back: %s\n", ((result = hdfsRename(fs, dstPath, srcPath)) ? "Failed!" : "Success!"));
totalResult += result;
const char* slashTmp = "/tmp";
const char* newDirectory = "/tmp/newdir";
fprintf(stderr, "hdfsCreateDirectory: %s\n", ((result = hdfsCreateDirectory(fs, newDirectory)) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr, "hdfsSetReplication: %s\n", ((result = hdfsSetReplication(fs, srcPath, 1)) ? "Failed!" : "Success!"));
totalResult += result;
char buffer[256];
const char *resp;
fprintf(stderr, "hdfsGetWorkingDirectory: %s\n", ((resp = hdfsGetWorkingDirectory(fs, buffer, sizeof(buffer))) ? buffer : "Failed!"));
totalResult += (resp ? 0 : 1);
fprintf(stderr, "hdfsSetWorkingDirectory: %s\n", ((result = hdfsSetWorkingDirectory(fs, slashTmp)) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr, "hdfsGetWorkingDirectory: %s\n", ((resp = hdfsGetWorkingDirectory(fs, buffer, sizeof(buffer))) ? buffer : "Failed!"));
totalResult += (resp ? 0 : 1);
hdfsFileInfo *fileInfo = NULL;
if((fileInfo = hdfsGetPathInfo(fs, slashTmp)) != NULL) {
fprintf(stderr, "hdfsGetPathInfo - SUCCESS!\n");
fprintf(stderr, "Name: %s, ", fileInfo->mName);
fprintf(stderr, "Type: %c, ", (char)(fileInfo->mKind));
fprintf(stderr, "Replication: %d, ", fileInfo->mReplication);
fprintf(stderr, "BlockSize: %lld, ", fileInfo->mBlockSize);
fprintf(stderr, "Size: %lld, ", fileInfo->mSize);
fprintf(stderr, "LastMod: %s", ctime(&fileInfo->mLastMod));
fprintf(stderr, "Owner: %s, ", fileInfo->mOwner);
fprintf(stderr, "Group: %s, ", fileInfo->mGroup);
char permissions[10];
permission_disp(fileInfo->mPermissions, permissions);
fprintf(stderr, "Permissions: %d (%s)\n", fileInfo->mPermissions, permissions);
hdfsFreeFileInfo(fileInfo, 1);
} else {
totalResult++;
fprintf(stderr, "waah! hdfsGetPathInfo for %s - FAILED!\n", slashTmp);
}
hdfsFileInfo *fileList = 0;
int numEntries = 0;
if((fileList = hdfsListDirectory(fs, slashTmp, &numEntries)) != NULL) {
int i = 0;
for(i=0; i < numEntries; ++i) {
fprintf(stderr, "Name: %s, ", fileList[i].mName);
fprintf(stderr, "Type: %c, ", (char)fileList[i].mKind);
fprintf(stderr, "Replication: %d, ", fileList[i].mReplication);
fprintf(stderr, "BlockSize: %lld, ", fileList[i].mBlockSize);
fprintf(stderr, "Size: %lld, ", fileList[i].mSize);
fprintf(stderr, "LastMod: %s", ctime(&fileList[i].mLastMod));
fprintf(stderr, "Owner: %s, ", fileList[i].mOwner);
fprintf(stderr, "Group: %s, ", fileList[i].mGroup);
char permissions[10];
permission_disp(fileList[i].mPermissions, permissions);
fprintf(stderr, "Permissions: %d (%s)\n", fileList[i].mPermissions, permissions);
}
hdfsFreeFileInfo(fileList, numEntries);
} else {
if (errno) {
totalResult++;
fprintf(stderr, "waah! hdfsListDirectory - FAILED!\n");
} else {
fprintf(stderr, "Empty directory!\n");
}
}
// char*** hosts = hdfsGetHosts(fs, srcPath, 0, 1);
// if(hosts) {
// fprintf(stderr, "hdfsGetHosts - SUCCESS! ... \n");
// int i=0;
// while(hosts[i]) {
// int j = 0;
// while(hosts[i][j]) {
// fprintf(stderr,
// "\thosts[%d][%d] - %s\n", i, j, hosts[i][j]);
// ++j;
// }
// ++i;
// }
// } else {
// totalResult++;
// fprintf(stderr, "waah! hdfsGetHosts - FAILED!\n");
// }
char *newOwner = "root";
// setting tmp dir to 777 so later when connectAsUser nobody, we can write to it
short newPerm = 0666;
// chown write
fprintf(stderr, "hdfsChown: %s\n", ((result = hdfsChown(fs, writePath, NULL, "users")) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr, "hdfsChown: %s\n", ((result = hdfsChown(fs, writePath, newOwner, NULL)) ? "Failed!" : "Success!"));
totalResult += result;
// chmod write
fprintf(stderr, "hdfsChmod: %s\n", ((result = hdfsChmod(fs, writePath, newPerm)) ? "Failed!" : "Success!"));
totalResult += result;
sleep(2);
tTime newMtime = time(NULL);
tTime newAtime = time(NULL);
// utime write
fprintf(stderr, "hdfsUtime: %s\n", ((result = hdfsUtime(fs, writePath, newMtime, newAtime)) ? "Failed!" : "Success!"));
totalResult += result;
// chown/chmod/utime read
hdfsFileInfo *finfo = hdfsGetPathInfo(fs, writePath);
fprintf(stderr, "hdfsChown read: %s\n", ((result = (strcmp(finfo->mOwner, newOwner) != 0)) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr, "hdfsChmod read: %s\n", ((result = (finfo->mPermissions != newPerm)) ? "Failed!" : "Success!"));
totalResult += result;
// will later use /tmp/ as a different user so enable it
fprintf(stderr, "hdfsChmod: %s\n", ((result = hdfsChmod(fs, "/tmp/", 0777)) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr,"newMTime=%ld\n",newMtime);
fprintf(stderr,"curMTime=%ld\n",finfo->mLastMod);
fprintf(stderr, "hdfsUtime read (mtime): %s\n", ((result = (finfo->mLastMod != newMtime / 1000)) ? "Failed!" : "Success!"));
totalResult += result;
hdfsFreeFileInfo(finfo, 1);
// Clean up
fprintf(stderr, "hdfsDelete: %s\n", ((result = hdfsDelete(fs, newDirectory, 1)) ? "Failed!" : "Success!"));
totalResult += result;
fprintf(stderr, "hdfsDelete: %s\n", ((result = hdfsDelete(fs, srcPath, 1)) ? "Failed!" : "Success!"));
totalResult += result;
// fprintf(stderr, "hdfsDelete: %s\n", ((result = hdfsDelete(fs, movePath, 1)) ? "Failed!" : "Success!"));
// totalResult += result;
fprintf(stderr, "hdfsExists: %s\n", ((result = hdfsExists(fs, newDirectory)) ? "Success!" : "Failed!"));
totalResult += (result ? 0 : 1);
}
{
// TEST APPENDS
const char *writePath = "/tmp/appends";
// CREATE
hdfsFile writeFile = hdfsOpenFile(fs, writePath, O_WRONLY, 0, 0, 0);
if(!writeFile) {
fprintf(stderr, "Failed to open %s for writing!\n", writePath);
exit(-1);
}
fprintf(stderr, "Opened %s for writing successfully...\n", writePath);
const char* buffer = "Hello,";
tSize num_written_bytes = hdfsWrite(fs, writeFile, (void*)buffer, strlen(buffer));
fprintf(stderr, "Wrote %d bytes\n", num_written_bytes);
if (hdfsFlush(fs, writeFile)) {
fprintf(stderr, "Failed to 'flush' %s\n", writePath);
exit(-1);
}
fprintf(stderr, "Flushed %s successfully!\n", writePath);
hdfsCloseFile(fs, writeFile);
fprintf(stderr, "hdfsSetReplication: %s\n", ((result = hdfsSetReplication(fs, writePath, 1)) ? "Failed!" : "Success!"));
totalResult += result;
// RE-OPEN
writeFile = hdfsOpenFile(fs, writePath, O_WRONLY|O_APPEND, 0, 0, 0);
if(!writeFile) {
fprintf(stderr, "Failed to open %s for writing!\n", writePath);
exit(-1);
}
fprintf(stderr, "Opened %s for appending successfully...\n", writePath);
buffer = " World";
num_written_bytes = hdfsWrite(fs, writeFile, (void*)buffer, strlen(buffer) + 1);
fprintf(stderr, "Wrote %d bytes\n", num_written_bytes);
if (hdfsFlush(fs, writeFile)) {
fprintf(stderr, "Failed to 'flush' %s\n", writePath);
exit(-1);
}
fprintf(stderr, "Flushed %s successfully!\n", writePath);
hdfsCloseFile(fs, writeFile);
// CHECK size
hdfsFileInfo *finfo = hdfsGetPathInfo(fs, writePath);
fprintf(stderr, "fileinfo->mSize: == total %s\n", ((result = (finfo->mSize == strlen("Hello, World") + 1)) ? "Success!" : "Failed!"));
totalResult += (result ? 0 : 1);
// READ and check data
hdfsFile readFile = hdfsOpenFile(fs, writePath, O_RDONLY, 0, 0, 0);
if (!readFile) {
fprintf(stderr, "Failed to open %s for reading!\n", writePath);
exit(-1);
}
char rdbuffer[32];
tSize num_read_bytes = hdfsRead(fs, readFile, (void*)rdbuffer, sizeof(rdbuffer));
fprintf(stderr, "Read following %d bytes:\n%s\n",
num_read_bytes, rdbuffer);
fprintf(stderr, "read == Hello, World %s\n", (result = (strcmp(rdbuffer, "Hello, World") == 0)) ? "Success!" : "Failed!");
hdfsCloseFile(fs, readFile);
// DONE test appends
}
totalResult += (hdfsDisconnect(fs) != 0);
{
//
// Now test as connecting as a specific user
// This is only meant to test that we connected as that user, not to test
// the actual fs user capabilities. Thus just create a file and read
// the owner is correct.
const char *tuser = "nobody";
const char* writePath = "/tmp/usertestfile.txt";
fs = hdfsConnectAsUserNewInstance("default", 50070, tuser);
if(!fs) {
fprintf(stderr, "Oops! Failed to connect to hdfs as user %s!\n",tuser);
exit(-1);
}
hdfsFile writeFile = hdfsOpenFile(fs, writePath, O_WRONLY|O_CREAT, 0, 0, 0);
if(!writeFile) {
fprintf(stderr, "Failed to open %s for writing!\n", writePath);
exit(-1);
}
fprintf(stderr, "Opened %s for writing successfully...\n", writePath);
char* buffer = "Hello, World!";
tSize num_written_bytes = hdfsWrite(fs, writeFile, (void*)buffer, strlen(buffer)+1);
fprintf(stderr, "Wrote %d bytes\n", num_written_bytes);
if (hdfsFlush(fs, writeFile)) {
fprintf(stderr, "Failed to 'flush' %s\n", writePath);
exit(-1);
}
fprintf(stderr, "Flushed %s successfully!\n", writePath);
hdfsCloseFile(fs, writeFile);
hdfsFileInfo *finfo = hdfsGetPathInfo(fs, writePath);
if (finfo) {
fprintf(stderr, "hdfs new file user is correct: %s\n", ((result = (strcmp(finfo->mOwner, tuser) != 0)) ? "Failed!" : "Success!"));
} else {
fprintf(stderr, "hdfsFileInfo returned by hdfsGetPathInfo is NULL\n");
result = -1;
}
totalResult += result;
}
totalResult += (hdfsDisconnect(fs) != 0);
fprintf(stderr, "totalResult == %d\n", totalResult);
if (totalResult != 0) {
return -1;
} else {
return 0;
}
}
/**
* vim: ts=4: sw=4: et:
*/

View File

@ -0,0 +1,73 @@
/**
* 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 "webhdfs.h"
#include <stdio.h>
#include <stdlib.h>
int main(int argc, char **argv) {
if (argc != 4) {
fprintf(stderr, "Usage: hdfs_read <filename> <filesize> <buffersize>\n");
exit(-1);
}
hdfsFS fs = hdfsConnect("0.0.0.0", 50070);
if (!fs) {
fprintf(stderr, "Oops! Failed to connect to hdfs!\n");
exit(-1);
}
const char* rfile = argv[1];
tSize fileTotalSize = strtoul(argv[2], NULL, 10);
tSize bufferSize = strtoul(argv[3], NULL, 10);
hdfsFile readFile = hdfsOpenFile(fs, rfile, O_RDONLY, bufferSize, 0, 0);
if (!readFile) {
fprintf(stderr, "Failed to open %s for writing!\n", rfile);
exit(-2);
}
// data to be written to the file
char* buffer = malloc(sizeof(char) * bufferSize);
if(buffer == NULL) {
return -2;
}
// read from the file
tSize curSize = bufferSize;
tSize totalReadSize = 0;
for (; (curSize = hdfsRead(fs, readFile, (void*)buffer, bufferSize)) == bufferSize ;) {
totalReadSize += curSize;
}
totalReadSize += curSize;
fprintf(stderr, "size of the file: %d; reading size: %d\n", fileTotalSize, totalReadSize);
free(buffer);
hdfsCloseFile(fs, readFile);
hdfsDisconnect(fs);
return 0;
}
/**
* vim: ts=4: sw=4: et:
*/

View File

@ -0,0 +1,225 @@
/**
* 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 "expect.h"
#include "webhdfs.h"
#include <errno.h>
#include <semaphore.h>
#include <pthread.h>
#include <stdio.h>
#include <stdlib.h>
#include <string.h>
#define TLH_MAX_THREADS 100
static sem_t *tlhSem;
static const char *nn;
static const char *user;
static int port;
struct tlhThreadInfo {
/** Thread index */
int threadIdx;
/** 0 = thread was successful; error code otherwise */
int success;
/** pthread identifier */
pthread_t thread;
};
static int hdfsSingleNameNodeConnect(const char *nn, int port, const char *user, hdfsFS *fs)
{
hdfsFS hdfs;
if (port < 0) {
fprintf(stderr, "hdfsSingleNameNodeConnect: nmdGetNameNodePort "
"returned error %d\n", port);
return port;
}
hdfs = hdfsConnectAsUserNewInstance(nn, port, user);
if (!hdfs) {
return -errno;
}
*fs = hdfs;
return 0;
}
static int doTestHdfsOperations(struct tlhThreadInfo *ti, hdfsFS fs)
{
char prefix[256], tmp[256];
hdfsFile file;
int ret, expected;
snprintf(prefix, sizeof(prefix), "/tlhData%04d", ti->threadIdx);
if (hdfsExists(fs, prefix) == 0) {
EXPECT_ZERO(hdfsDelete(fs, prefix, 1));
}
EXPECT_ZERO(hdfsCreateDirectory(fs, prefix));
snprintf(tmp, sizeof(tmp), "%s/file", prefix);
/*
* Although there should not be any file to open for reading,
* the right now implementation only construct a local
* information struct when opening file
*/
EXPECT_NONNULL(hdfsOpenFile(fs, tmp, O_RDONLY, 0, 0, 0));
file = hdfsOpenFile(fs, tmp, O_WRONLY, 0, 0, 0);
EXPECT_NONNULL(file);
/* TODO: implement writeFully and use it here */
expected = strlen(prefix);
ret = hdfsWrite(fs, file, prefix, expected);
if (ret < 0) {
ret = errno;
fprintf(stderr, "hdfsWrite failed and set errno %d\n", ret);
return ret;
}
if (ret != expected) {
fprintf(stderr, "hdfsWrite was supposed to write %d bytes, but "
"it wrote %d\n", ret, expected);
return EIO;
}
EXPECT_ZERO(hdfsFlush(fs, file));
EXPECT_ZERO(hdfsCloseFile(fs, file));
/* Let's re-open the file for reading */
file = hdfsOpenFile(fs, tmp, O_RDONLY, 0, 0, 0);
EXPECT_NONNULL(file);
/* TODO: implement readFully and use it here */
ret = hdfsRead(fs, file, tmp, sizeof(tmp));
if (ret < 0) {
ret = errno;
fprintf(stderr, "hdfsRead failed and set errno %d\n", ret);
return ret;
}
if (ret != expected) {
fprintf(stderr, "hdfsRead was supposed to read %d bytes, but "
"it read %d\n", ret, expected);
return EIO;
}
EXPECT_ZERO(memcmp(prefix, tmp, expected));
EXPECT_ZERO(hdfsCloseFile(fs, file));
// TODO: Non-recursive delete should fail?
//EXPECT_NONZERO(hdfsDelete(fs, prefix, 0));
EXPECT_ZERO(hdfsDelete(fs, prefix, 1));
return 0;
}
static void *testHdfsOperations(void *v)
{
struct tlhThreadInfo *ti = (struct tlhThreadInfo*)v;
hdfsFS fs = NULL;
int ret;
fprintf(stderr, "testHdfsOperations(threadIdx=%d): starting\n",
ti->threadIdx);
ret = hdfsSingleNameNodeConnect(nn, port, user, &fs);
if (ret) {
fprintf(stderr, "testHdfsOperations(threadIdx=%d): "
"hdfsSingleNameNodeConnect failed with error %d.\n",
ti->threadIdx, ret);
ti->success = EIO;
return NULL;
}
ti->success = doTestHdfsOperations(ti, fs);
if (hdfsDisconnect(fs)) {
ret = errno;
fprintf(stderr, "hdfsDisconnect error %d\n", ret);
ti->success = ret;
}
return NULL;
}
static int checkFailures(struct tlhThreadInfo *ti, int tlhNumThreads)
{
int i, threadsFailed = 0;
const char *sep = "";
for (i = 0; i < tlhNumThreads; i++) {
if (ti[i].success != 0) {
threadsFailed = 1;
}
}
if (!threadsFailed) {
fprintf(stderr, "testLibHdfs: all threads succeeded. SUCCESS.\n");
return EXIT_SUCCESS;
}
fprintf(stderr, "testLibHdfs: some threads failed: [");
for (i = 0; i < tlhNumThreads; i++) {
if (ti[i].success != 0) {
fprintf(stderr, "%s%d", sep, i);
sep = ", ";
}
}
fprintf(stderr, "]. FAILURE.\n");
return EXIT_FAILURE;
}
/**
* Test that we can write a file with libhdfs and then read it back
*/
int main(int argc, const char *args[])
{
if (argc != 4) {
fprintf(stderr, "usage: test_libhdfs_threaded <namenode> <port> <username>");
return -1;
}
nn = args[1];
port = atoi(args[2]);
user = args[3];
int i, tlhNumThreads;
const char *tlhNumThreadsStr;
struct tlhThreadInfo ti[TLH_MAX_THREADS];
tlhNumThreadsStr = getenv("TLH_NUM_THREADS");
if (!tlhNumThreadsStr) {
tlhNumThreadsStr = "3";
}
tlhNumThreads = atoi(tlhNumThreadsStr);
if ((tlhNumThreads <= 0) || (tlhNumThreads > TLH_MAX_THREADS)) {
fprintf(stderr, "testLibHdfs: must have a number of threads "
"between 1 and %d inclusive, not %d\n",
TLH_MAX_THREADS, tlhNumThreads);
return EXIT_FAILURE;
}
memset(&ti[0], 0, sizeof(ti));
for (i = 0; i < tlhNumThreads; i++) {
ti[i].threadIdx = i;
}
// tlhSem = sem_open("sem", O_CREAT, 0644, tlhNumThreads);
for (i = 0; i < tlhNumThreads; i++) {
EXPECT_ZERO(pthread_create(&ti[i].thread, NULL,
testHdfsOperations, &ti[i]));
}
for (i = 0; i < tlhNumThreads; i++) {
EXPECT_ZERO(pthread_join(ti[i].thread, NULL));
}
// EXPECT_ZERO(sem_close(tlhSem));
return checkFailures(ti, tlhNumThreads);
}

View File

@ -0,0 +1,118 @@
/**
* 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 "webhdfs.h"
#include <limits.h>
#include <stdio.h>
#include <stdlib.h>
#include <string.h>
int main(int argc, char **argv) {
if (argc != 6) {
fprintf(stderr, "Usage: hdfs_write <filename> <filesize> <buffersize> <username> <append>\n");
exit(-1);
}
hdfsFS fs = hdfsConnectAsUser("0.0.0.0", 50070, argv[4]);
if (!fs) {
fprintf(stderr, "Oops! Failed to connect to hdfs!\n");
exit(-1);
}
const char* writeFileName = argv[1];
off_t fileTotalSize = strtoul(argv[2], NULL, 10);
long long tmpBufferSize = strtoul(argv[3], NULL, 10);
// sanity check
if(fileTotalSize == ULONG_MAX && errno == ERANGE) {
fprintf(stderr, "invalid file size %s - must be <= %lu\n", argv[2], ULONG_MAX);
exit(-3);
}
// currently libhdfs writes are of tSize which is int32
if(tmpBufferSize > INT_MAX) {
fprintf(stderr, "invalid buffer size libhdfs API write chunks must be <= %d\n",INT_MAX);
exit(-3);
}
tSize bufferSize = tmpBufferSize;
hdfsFile writeFile = NULL;
int append = atoi(argv[5]);
if (!append) {
writeFile = hdfsOpenFile(fs, writeFileName, O_WRONLY, bufferSize, 2, 0);
} else {
writeFile = hdfsOpenFile(fs, writeFileName, O_WRONLY | O_APPEND, bufferSize, 2, 0);
}
if (!writeFile) {
fprintf(stderr, "Failed to open %s for writing!\n", writeFileName);
exit(-2);
}
// data to be written to the file
char* buffer = malloc(sizeof(char) * bufferSize + 1);
if(buffer == NULL) {
fprintf(stderr, "Could not allocate buffer of size %d\n", bufferSize);
return -2;
}
int i = 0;
for (i=0; i < bufferSize; ++i) {
buffer[i] = 'a' + (i%26);
}
buffer[bufferSize] = '\0';
size_t totalWriteSize = 0;
for (; totalWriteSize < fileTotalSize; ) {
tSize toWrite = bufferSize < (fileTotalSize - totalWriteSize) ? bufferSize : (fileTotalSize - totalWriteSize);
size_t written = hdfsWrite(fs, writeFile, (void*)buffer, toWrite);
fprintf(stderr, "written size %ld, to write size %d\n", written, toWrite);
totalWriteSize += written;
//sleep(1);
}
free(buffer);
hdfsCloseFile(fs, writeFile);
fprintf(stderr, "file total size: %lld, total write size: %ld\n", fileTotalSize, totalWriteSize);
hdfsFile readFile = hdfsOpenFile(fs, writeFileName, O_RDONLY, 0, 0, 0);
//sleep(1);
fprintf(stderr, "hdfsAvailable: %d\n", hdfsAvailable(fs, readFile));
hdfsFile writeFile2 = hdfsOpenFile(fs, writeFileName, O_WRONLY | O_APPEND, 0, 2, 0);
fprintf(stderr, "Opened %s for writing successfully...\n", writeFileName);
const char *content = "Hello, World!";
size_t num_written_bytes = hdfsWrite(fs, writeFile2, content, strlen(content) + 1);
if (num_written_bytes != strlen(content) + 1) {
fprintf(stderr, "Failed to write correct number of bytes - expected %d, got %d\n",
(int)(strlen(content) + 1), (int)num_written_bytes);
exit(-1);
}
fprintf(stderr, "Wrote %zd bytes\n", num_written_bytes);
hdfsDisconnect(fs);
return 0;
}
/**
* vim: ts=4: sw=4: et:
*/

View File

@ -0,0 +1,110 @@
#include <time.h>
#include <stdio.h>
#include <stdlib.h>
#include <sys/time.h>
#include "webhdfs.h"
#ifdef __MACH__
#include <mach/clock.h>
#include <mach/mach.h>
#endif
void current_utc_time(struct timespec *ts) {
#ifdef __MACH__ // OS X does not have clock_gettime, use clock_get_time
clock_serv_t cclock;
mach_timespec_t mts;
host_get_clock_service(mach_host_self(), CALENDAR_CLOCK, &cclock);
clock_get_time(cclock, &mts);
mach_port_deallocate(mach_task_self(), cclock);
ts->tv_sec = mts.tv_sec;
ts->tv_nsec = mts.tv_nsec;
#else
clock_gettime(CLOCK_REALTIME, ts);
#endif
}
long get_time() {
struct timespec tp;
current_utc_time(&tp);
return (long)((tp.tv_sec * 1000000000) + tp.tv_nsec);
}
#define SIZE 512*1024*1024
#define READ_SIZE 512*1024*1024
#define DISCARD_COUNT 5
int main(int argc, char** argv) {
if (argc != 4) {
fprintf(stderr, "Usage: test_read_bm <namenode> <user_name> <iteration_number>\n");
exit(0);
}
hdfsFS fs = hdfsConnectAsUser(argv[1], 50070, argv[2]);
/* printf("File is null: %d\n", file == NULL ? 1 : 0); */
char *buf = (char *) malloc(sizeof(unsigned char) * SIZE);
printf("Read size: %d\n", READ_SIZE);
int iterations = atoi(argv[3]);
if (iterations <= DISCARD_COUNT) {
printf("Iterations should be at least %d\n", DISCARD_COUNT + 1);
exit(0);
}
printf("Running %d iterations\n", iterations);
float time_total;
float max = 0.f;
float min = 999999999999999.f;
printf("Start...\n");
int i;
for (i=0; i<iterations; ++i) {
long start = get_time();
hdfsFile file = hdfsOpenFile(fs, "/tmp/512_mb.txt", O_RDONLY, 0, 0, 0);
int n = 0;
while (n < SIZE) {
int nread = hdfsRead(fs, file, buf + n, READ_SIZE);
if (nread <= 0) {
printf("EOF before finished, read %d bytes\n", n);
hdfsDisconnect(fs);
return 0;
}
n += nread;
printf("Read %d kilobytes\n", nread / 1024);
}
long end = get_time();
printf("Read %d bytes, hoping for %d.\n", n, SIZE);
long elapsed = (end - start);
printf("Start: %lu, end: %lu\n", start, end);
float time = elapsed / (1000000000.0f);
printf ("Took %2.6fs\n", time);
printf("Throughput: %2.2fMB/s\n", SIZE * 1.0f / (1024 * 1024 * time));
if (i >= DISCARD_COUNT) {
time_total += time;
if (time < min) {
min = time;
}
if (time > max) {
max = time;
}
}
}
hdfsDisconnect(fs);
printf("------\n");
printf("Average time: %2.2fs\n", time_total / (iterations - DISCARD_COUNT));
printf("Max. time: %2.2f, min. time: %2.2f\n", max, min);
float maxt = SIZE * 1.f / (1024 * 1024 * max);
float mint = SIZE * 1.f / (1024 * 1024 * min);
printf("Average throughput: %2.2fMB/s\n", 1.f * SIZE * (iterations - DISCARD_COUNT) / (1024 * 1024 * time_total));
printf("Max. throughput: %2.2f, min. throughput: %2.2f\n", maxt, mint);
// printf("File contents: %d\n", buf[0]);
return 0;
}

View File

@ -0,0 +1,694 @@
/**
* 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 LIB_WEBHDFS_H
#define LIB_WEBHDFS_H
#include <errno.h> /* for EINTERNAL, etc. */
#include <fcntl.h> /* for O_RDONLY, O_WRONLY */
#include <stdint.h> /* for uint64_t, etc. */
#include <time.h> /* for time_t */
#include <pthread.h>
#ifndef O_RDONLY
#define O_RDONLY 1
#endif
#ifndef O_WRONLY
#define O_WRONLY 2
#endif
#ifndef EINTERNAL
#define EINTERNAL 255
#endif
/** All APIs set errno to meaningful values */
#ifdef __cplusplus
extern "C" {
#endif
/**
* Some utility decls used in libhdfs.
*/
typedef int32_t tSize; /// size of data for read/write io ops
typedef time_t tTime; /// time type in seconds
typedef int64_t tOffset;/// offset within the file
typedef uint16_t tPort; /// port
/**
* The information required for accessing webhdfs,
* including the network address of the namenode and the user name
*/
struct hdfsBuilder {
int forceNewInstance;
const char *nn;
const char *nn_jni;
tPort port;
const char *kerbTicketCachePath;
const char *userName;
/*
* This is a new attribute compared to libhdfs.
* We maintain a local workingDir for constructing absolute path
*/
char *workingDir;
};
typedef enum tObjectKind {
kObjectKindFile = 'F',
kObjectKindDirectory = 'D',
} tObjectKind;
/**
* For libhdfs based on JNI, this is used as
* the C reflection of org.apache.org.hadoop.FileSystem .
* In the current libwebhdfs based on webhdfs,
* this is actually hdfsBuilder which contains
* the network address of the namenode and the user name
*/
struct hdfs_internal;
typedef struct hdfs_internal* hdfsFS;
/**
* The C equivalent of org.apache.org.hadoop.FSData(Input|Output)Stream .
*/
enum hdfsStreamType
{
UNINITIALIZED = 0,
INPUT = 1,
OUTPUT = 2,
};
/**
* The 'file-handle' to a file in hdfs.
*/
struct hdfsFile_internal {
void* file;
enum hdfsStreamType type;
int flags;
tOffset offset;
};
typedef struct hdfsFile_internal* hdfsFile;
/**
* hdfsFileInfo - Information about a file/directory.
*/
typedef struct {
tObjectKind mKind; /* file or directory */
char *mName; /* the name of the file */
tTime mLastMod; /* the last modification time for the file in seconds */
tOffset mSize; /* the size of the file in bytes */
short mReplication; /* the count of replicas */
tOffset mBlockSize; /* the block size for the file */
char *mOwner; /* the owner of the file */
char *mGroup; /* the group associated with the file */
short mPermissions; /* the permissions associated with the file */
tTime mLastAccess; /* the last access time for the file in seconds */
} hdfsFileInfo;
/**
* webhdfsBuffer - used for hold the data for read/write from/to http connection
*/
typedef struct {
const char *wbuffer; /* the user's buffer for uploading */
size_t remaining; /* length of content */
size_t offset; /* offset for reading */
int openFlag; /* check whether the hdfsOpenFile has been called before */
int closeFlag; /* whether to close the http connection for writing */
pthread_mutex_t writeMutex; // used for syschronization between the curl thread and the hdfsWrite thread
pthread_cond_t newwrite_or_close; // transferring thread waits for this condition
// when there is no more content for transferring in the buffer
pthread_cond_t transfer_finish; // condition used to indicate finishing transferring (one buffer)
} webhdfsBuffer;
struct webhdfsFileHandle {
char *absPath;
int bufferSize;
short replication;
tSize blockSize;
char *datanode;
webhdfsBuffer *uploadBuffer;
pthread_t connThread;
};
// Bit fields for hdfsFile_internal flags
#define HDFS_FILE_SUPPORTS_DIRECT_READ (1<<0)
/**
* Determine if a file is open for read.
*
* @param file The HDFS file
* @return 1 if the file is open for read; 0 otherwise
*/
int hdfsFileIsOpenForRead(hdfsFile file);
/**
* Determine if a file is open for write.
*
* @param file The HDFS file
* @return 1 if the file is open for write; 0 otherwise
*/
int hdfsFileIsOpenForWrite(hdfsFile file);
/**
* Disable the direct read optimization for a file in libhdfs.
* This is mainly provided for unit testing purposes.
* No longer useful in libwebhdfs since libwebhdfs is based on webhdfs.
*
* @param file The HDFS file
*/
void hdfsFileDisableDirectRead(hdfsFile file);
/**
* hdfsConnectAsUser - Connect to a hdfs file system as a specific user
* Connect to the hdfs.
* @param nn The NameNode. See hdfsBuilderSetNameNode for details.
* @param port The port on which the server is listening.
* @param user the user name (this is hadoop domain user). Or NULL is equivelant to hhdfsConnect(host, port)
* @return Returns a handle to the filesystem or NULL on error.
* @deprecated Use hdfsBuilderConnect instead.
*/
hdfsFS hdfsConnectAsUser(const char* nn, tPort port, const char *user);
/**
* hdfsConnect - Connect to a hdfs file system.
* Connect to the hdfs.
* @param nn The NameNode. See hdfsBuilderSetNameNode for details.
* @param port The port on which the server is listening.
* @return Returns a handle to the filesystem or NULL on error.
* @deprecated Use hdfsBuilderConnect instead.
*/
hdfsFS hdfsConnect(const char* nn, tPort port);
/**
* hdfsConnect - Connect to an hdfs file system.
*
* The effect with hdfsConnectAsUser in libwebhdfs.
*
* @param nn The NameNode. See hdfsBuilderSetNameNode for details.
* @param port The port on which the server is listening.
* @param user The user name to use when connecting
* @return Returns a handle to the filesystem or NULL on error.
* @deprecated Use hdfsBuilderConnect instead.
*/
hdfsFS hdfsConnectAsUserNewInstance(const char* nn, tPort port, const char *user );
/**
* hdfsConnect - Connect to an hdfs file system.
*
* The same effect with hdfsConnect in libwebhdfs.
*
* @param nn The NameNode. See hdfsBuilderSetNameNode for details.
* @param port The port on which the server is listening.
* @return Returns a handle to the filesystem or NULL on error.
* @deprecated Use hdfsBuilderConnect instead.
*/
hdfsFS hdfsConnectNewInstance(const char* nn, tPort port);
/**
* Connect to HDFS using the parameters defined by the builder.
*
* Every successful call to hdfsBuilderConnect should be matched with a call
* to hdfsDisconnect, when the hdfsFS is no longer needed.
*
* @param bld The HDFS builder
* @return Returns a handle to the filesystem, or NULL on error.
*/
hdfsFS hdfsBuilderConnect(struct hdfsBuilder *bld);
/**
* Create an HDFS builder.
*
* @return The HDFS builder, or NULL on error.
*/
struct hdfsBuilder *hdfsNewBuilder(void);
/**
* In libhdfs: force the builder to always create a new instance of the FileSystem,
* rather than possibly finding one in the cache.
*
* @param bld The HDFS builder
* @deprecated No longer usefule in libwebhdfs.
*/
void hdfsBuilderSetForceNewInstance(struct hdfsBuilder *bld);
/**
* Set the HDFS NameNode to connect to.
*
* @param bld The HDFS builder
* @param nn The NameNode to use.
*
* If the string given is 'default', the default NameNode
* configuration will be used (from the XML configuration files)
*
* If NULL is given, a LocalFileSystem will be created.
*
* If the string starts with a protocol type such as file:// or
* hdfs://, this protocol type will be used. If not, the
* hdfs:// protocol type will be used.
*
* You may specify a NameNode port in the usual way by
* passing a string of the format hdfs://<hostname>:<port>.
* Alternately, you may set the port with
* hdfsBuilderSetNameNodePort. However, you must not pass the
* port in two different ways.
*/
void hdfsBuilderSetNameNode(struct hdfsBuilder *bld, const char *nn);
/**
* Set the port of the HDFS NameNode to connect to.
*
* @param bld The HDFS builder
* @param port The port.
*/
void hdfsBuilderSetNameNodePort(struct hdfsBuilder *bld, tPort port);
/**
* Set the username to use when connecting to the HDFS cluster.
*
* @param bld The HDFS builder
* @param userName The user name. The string will be shallow-copied.
*/
void hdfsBuilderSetUserName(struct hdfsBuilder *bld, const char *userName);
/**
* Set the path to the Kerberos ticket cache to use when connecting to
* the HDFS cluster.
*
* @param bld The HDFS builder
* @param kerbTicketCachePath The Kerberos ticket cache path. The string
* will be shallow-copied.
*/
void hdfsBuilderSetKerbTicketCachePath(struct hdfsBuilder *bld,
const char *kerbTicketCachePath);
/**
* Free an HDFS builder.
*
* @param bld The HDFS builder
*/
void hdfsFreeBuilder(struct hdfsBuilder *bld);
/**
* Get a configuration string.
*
* @param key The key to find
* @param val (out param) The value. This will be set to NULL if the
* key isn't found. You must free this string with
* hdfsConfStrFree.
*
* @return 0 on success; nonzero error code otherwise.
* Failure to find the key is not an error.
*/
int hdfsConfGetStr(const char *key, char **val);
/**
* Get a configuration integer.
*
* @param key The key to find
* @param val (out param) The value. This will NOT be changed if the
* key isn't found.
*
* @return 0 on success; nonzero error code otherwise.
* Failure to find the key is not an error.
*/
int hdfsConfGetInt(const char *key, int32_t *val);
/**
* Free a configuration string found with hdfsConfGetStr.
*
* @param val A configuration string obtained from hdfsConfGetStr
*/
void hdfsConfStrFree(char *val);
/**
* hdfsDisconnect - Disconnect from the hdfs file system.
* Disconnect from hdfs.
*
* In libwebhdfs, we simply free the hdfsFS,
* so do not use it after hdfsCopy/hdfsMove/hdfsGetDefaultBlockSize which still use JNI for FileSystem connection.
*
* @param fs The configured filesystem handle.
* @return Returns 0 on success, -1 on error.
*/
int hdfsDisconnect(hdfsFS fs);
/**
* hdfsOpenFile - Open a hdfs file in given mode.
* In libwebhdfs we simply store corresponding information in a hdfsFile.
*
* @param fs The configured filesystem handle.
* @param path The full path to the file.
* @param flags - an | of bits/fcntl.h file flags - supported flags are O_RDONLY, O_WRONLY (meaning create or overwrite i.e., implies O_TRUNCAT),
* O_WRONLY|O_APPEND. Other flags are generally ignored other than (O_RDWR || (O_EXCL & O_CREAT)) which return NULL and set errno equal ENOTSUP.
* @param bufferSize Size of buffer for read/write - pass 0 if you want
* to use the default configured values.
* @param replication Block replication - pass 0 if you want to use
* the default configured values.
* @param blocksize Size of block - pass 0 if you want to use the
* default configured values.
* @return Returns the handle to the open file or NULL on error.
*/
hdfsFile hdfsOpenFile(hdfsFS fs, const char* path, int flags,
int bufferSize, short replication, tSize blocksize);
/**
* hdfsCloseFile - Close an open file.
* @param fs The configured filesystem handle.
* @param file The file handle.
* @return Returns 0 on success, -1 on error.
*/
int hdfsCloseFile(hdfsFS fs, hdfsFile file);
/**
* hdfsExists - Checks if a given path exsits on the filesystem
* @param fs The configured filesystem handle.
* @param path The path to look for
* @return Returns 0 on success, -1 on error.
*/
int hdfsExists(hdfsFS fs, const char *path);
/**
* hdfsSeek - Seek to given offset in file.
* This works only for files opened in read-only mode.
* In libwebhdfs we store the offset in the local hdfsFile handle, thus
* in this function we simply set the local offset.
*
* @param fs The configured filesystem handle.
* @param file The file handle.
* @param desiredPos Offset into the file to seek into.
* @return Returns 0 on success, -1 on error.
*/
int hdfsSeek(hdfsFS fs, hdfsFile file, tOffset desiredPos);
/**
* hdfsTell - Get the current offset in the file, in bytes.
* In libwebhdfs the current offset is stored in the local hdfsFile handle,
* thus this function simply sets the local offset.
* @param fs The configured filesystem handle.
* @param file The file handle.
* @return Current offset, -1 on error.
*/
tOffset hdfsTell(hdfsFS fs, hdfsFile file);
/**
* hdfsRead - Read data from an open file.
* In libwebhdfs the reading starts from the current offset which is stored in the hdfsFile handle
* @param fs The configured filesystem handle.
* @param file The file handle.
* @param buffer The buffer to copy read bytes into.
* @param length The length of the buffer.
* @return On success, a positive number indicating how many bytes
* were read.
* On end-of-file, 0.
* On error, -1. Errno will be set to the error code.
* Just like the POSIX read function, hdfsRead will return -1
* and set errno to EINTR if data is temporarily unavailable,
* but we are not yet at the end of the file.
*/
tSize hdfsRead(hdfsFS fs, hdfsFile file, void* buffer, tSize length);
/**
* hdfsPread - Positional read of data from an open file.
* @param fs The configured filesystem handle.
* @param file The file handle.
* @param position Position from which to read
* @param buffer The buffer to copy read bytes into.
* @param length The length of the buffer.
* @return Returns the number of bytes actually read, possibly less than
* than length;-1 on error.
*/
tSize hdfsPread(hdfsFS fs, hdfsFile file, tOffset position,
void* buffer, tSize length);
/**
* hdfsWrite - Write data into an open file.
* @param fs The configured filesystem handle.
* @param file The file handle.
* @param buffer The data.
* @param length The no. of bytes to write.
* @return Returns the number of bytes written, -1 on error.
*/
tSize hdfsWrite(hdfsFS fs, hdfsFile file, const void* buffer,
tSize length);
/**
* hdfsWrite - Flush the data. No use for libwebhdfs.
* @param fs The configured filesystem handle.
* @param file The file handle.
* @return Returns 0 on success, -1 on error.
* @deprecated Not usefule in libwebhdfs.
*/
int hdfsFlush(hdfsFS fs, hdfsFile file);
/**
* hdfsHFlush - Flush out the data in client's user buffer. After the
* return of this call, new readers will see the data.
* @param fs configured filesystem handle
* @param file file handle
* @return 0 on success, -1 on error and sets errno
* @deprecated Not usefule in libwebhdfs.
*/
int hdfsHFlush(hdfsFS fs, hdfsFile file);
/**
* hdfsAvailable - Number of bytes that can be read from this
* input stream.
* @param fs The configured filesystem handle.
* @param file The file handle.
* @return Returns available bytes; -1 on error.
*/
int hdfsAvailable(hdfsFS fs, hdfsFile file);
/**
* hdfsCopy - Copy file from one filesystem to another.
* @param srcFS The handle to source filesystem.
* @param src The path of source file.
* @param dstFS The handle to destination filesystem.
* @param dst The path of destination file.
* @return Returns 0 on success, -1 on error.
*/
int hdfsCopy(hdfsFS srcFS, const char* src, hdfsFS dstFS, const char* dst);
/**
* hdfsMove - Move file from one filesystem to another.
* @param srcFS The handle to source filesystem.
* @param src The path of source file.
* @param dstFS The handle to destination filesystem.
* @param dst The path of destination file.
* @return Returns 0 on success, -1 on error.
*/
int hdfsMove(hdfsFS srcFS, const char* src, hdfsFS dstFS, const char* dst);
/**
* hdfsDelete - Delete file.
* @param fs The configured filesystem handle.
* @param path The path of the file.
* @param recursive if path is a directory and set to
* non-zero, the directory is deleted else throws an exception. In
* case of a file the recursive argument is irrelevant.
* @return Returns 0 on success, -1 on error.
*/
int hdfsDelete(hdfsFS fs, const char* path, int recursive);
/**
* hdfsRename - Rename file.
* @param fs The configured filesystem handle.
* @param oldPath The path of the source file.
* @param newPath The path of the destination file.
* @return Returns 0 on success, -1 on error.
*/
int hdfsRename(hdfsFS fs, const char* oldPath, const char* newPath);
/**
* hdfsGetWorkingDirectory - Get the current working directory for
* the given filesystem. In libwebhdfs it is retrieved from local hdfsFS handle.
* @param fs The configured filesystem handle.
* @param buffer The user-buffer to copy path of cwd into.
* @param bufferSize The length of user-buffer.
* @return Returns buffer, NULL on error.
*/
char* hdfsGetWorkingDirectory(hdfsFS fs, char *buffer, size_t bufferSize);
/**
* hdfsSetWorkingDirectory - Set the working directory. All relative
* paths will be resolved relative to it. In libwebhdfs the local hdfsFS is modified.
* @param fs The configured filesystem handle.
* @param path The path of the new 'cwd'.
* @return Returns 0 on success, -1 on error.
*/
int hdfsSetWorkingDirectory(hdfsFS fs, const char* path);
/**
* hdfsCreateDirectory - Make the given file and all non-existent
* parents into directories.
* @param fs The configured filesystem handle.
* @param path The path of the directory.
* @return Returns 0 on success, -1 on error.
*/
int hdfsCreateDirectory(hdfsFS fs, const char* path);
/**
* hdfsSetReplication - Set the replication of the specified
* file to the supplied value
* @param fs The configured filesystem handle.
* @param path The path of the file.
* @return Returns 0 on success, -1 on error.
*/
int hdfsSetReplication(hdfsFS fs, const char* path, int16_t replication);
/**
* hdfsListDirectory - Get list of files/directories for a given
* directory-path. hdfsFreeFileInfo should be called to deallocate memory.
* @param fs The configured filesystem handle.
* @param path The path of the directory.
* @param numEntries Set to the number of files/directories in path.
* @return Returns a dynamically-allocated array of hdfsFileInfo
* objects; NULL on error.
*/
hdfsFileInfo *hdfsListDirectory(hdfsFS fs, const char* path,
int *numEntries);
/**
* hdfsGetPathInfo - Get information about a path as a (dynamically
* allocated) single hdfsFileInfo struct. hdfsFreeFileInfo should be
* called when the pointer is no longer needed.
* @param fs The configured filesystem handle.
* @param path The path of the file.
* @return Returns a dynamically-allocated hdfsFileInfo object;
* NULL on error.
*/
hdfsFileInfo *hdfsGetPathInfo(hdfsFS fs, const char* path);
/**
* hdfsFreeFileInfo - Free up the hdfsFileInfo array (including fields)
* @param hdfsFileInfo The array of dynamically-allocated hdfsFileInfo
* objects.
* @param numEntries The size of the array.
*/
void hdfsFreeFileInfo(hdfsFileInfo *hdfsFileInfo, int numEntries);
/**
* hdfsGetHosts - Get hostnames where a particular block (determined by
* pos & blocksize) of a file is stored. The last element in the array
* is NULL. Due to replication, a single block could be present on
* multiple hosts.
* @param fs The configured filesystem handle.
* @param path The path of the file.
* @param start The start of the block.
* @param length The length of the block.
* @return Returns a dynamically-allocated 2-d array of blocks-hosts;
* NULL on error.
*
* Not supported yet but will be supported by libwebhdfs based on webhdfs.
*/
char*** hdfsGetHosts(hdfsFS fs, const char* path,
tOffset start, tOffset length);
/**
* hdfsFreeHosts - Free up the structure returned by hdfsGetHosts
* @param hdfsFileInfo The array of dynamically-allocated hdfsFileInfo
* objects.
* @param numEntries The size of the array.
*/
void hdfsFreeHosts(char ***blockHosts);
/**
* hdfsGetDefaultBlockSize - Get the optimum blocksize.
* @param fs The configured filesystem handle.
* @return Returns the blocksize; -1 on error.
*/
tOffset hdfsGetDefaultBlockSize(hdfsFS fs);
/**
* hdfsGetCapacity - Return the raw capacity of the filesystem.
* @param fs The configured filesystem handle.
* @return Returns the raw-capacity; -1 on error.
*
* Not supported yet but will be supported by libwebhdfs based on webhdfs.
*/
tOffset hdfsGetCapacity(hdfsFS fs);
/**
* hdfsGetUsed - Return the total raw size of all files in the filesystem.
* @param fs The configured filesystem handle.
* @return Returns the total-size; -1 on error.
*
* Not supported yet but will be supported by libwebhdfs based on webhdfs.
*/
tOffset hdfsGetUsed(hdfsFS fs);
/**
* hdfsChown
* @param fs The configured filesystem handle.
* @param path the path to the file or directory
* @param owner this is a string in Hadoop land. Set to null or "" if only setting group
* @param group this is a string in Hadoop land. Set to null or "" if only setting user
* @return 0 on success else -1
*/
int hdfsChown(hdfsFS fs, const char* path, const char *owner, const char *group);
/**
* hdfsChmod
* @param fs The configured filesystem handle.
* @param path the path to the file or directory
* @param mode the bitmask to set it to
* @return 0 on success else -1
*/
int hdfsChmod(hdfsFS fs, const char* path, short mode);
/**
* hdfsUtime
* @param fs The configured filesystem handle.
* @param path the path to the file or directory
* @param mtime new modification time or -1 for no change
* @param atime new access time or -1 for no change
* @return 0 on success else -1
*/
int hdfsUtime(hdfsFS fs, const char* path, tTime mtime, tTime atime);
#ifdef __cplusplus
}
#endif
#endif /*LIB_WEBHDFS_H*/