Modified: hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/SyncRequestProcessor.java URL: http://svn.apache.org/viewvc/hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/SyncRequestProcessor.java?rev=671303&r1=671302&r2=671303&view=diff ============================================================================== --- hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/SyncRequestProcessor.java (original) +++ hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/SyncRequestProcessor.java Tue Jun 24 12:04:58 2008 @@ -1,264 +1,264 @@ -/** - * 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. - */ - -package org.apache.zookeeper.server; - -import java.io.ByteArrayOutputStream; -import java.io.File; -import java.io.FileOutputStream; -import java.io.IOException; -import java.nio.ByteBuffer; -import java.nio.channels.FileChannel; -import java.util.LinkedList; -import java.util.Random; -import java.util.concurrent.LinkedBlockingQueue; - -import org.apache.log4j.Logger; - -import org.apache.jute.BinaryOutputArchive; -import org.apache.jute.Record; -import org.apache.zookeeper.server.util.Profiler; -import org.apache.zookeeper.txn.TxnHeader; - -/** - * This RequestProcessor logs requests to disk. It batches the requests to do - * the io efficiently. The request is not passed to the next RequestProcessor - * until its log has been synced to disk. - */ -public class SyncRequestProcessor extends Thread implements RequestProcessor { - private static final Logger LOG = Logger.getLogger(SyncRequestProcessor.class); - - static final int PADDING_TIMEOUT=1000; - ZooKeeperServer zks; - - LinkedBlockingQueue<Request> queuedRequests = new LinkedBlockingQueue<Request>(); - - static boolean forceSync; - static { - forceSync = !System.getProperty("zookeeper.forceSync", "yes").equals( - "no"); - } - - static long preAllocSize = 65536 * 1024; - static { - String size = System.getProperty("zookeeper.preAllocSize"); - if (size != null) { - try { - preAllocSize = Long.parseLong(size) * 1024; - } catch (NumberFormatException e) { - LOG.warn(size + " is not a valid value for preAllocSize"); - } - } - } - - /** - * The number of log entries to log before starting a snapshot - */ - static public int snapCount = ZooKeeperServer.getSnapCount(); - - Thread snapInProcess; - - RequestProcessor nextProcessor; - - boolean timeToDie = false; - - public SyncRequestProcessor(ZooKeeperServer zks, - RequestProcessor nextProcessor) { - super("SyncThread"); - this.zks = zks; - this.nextProcessor = nextProcessor; - start(); - } - - /** - * Transactions that have been written and are waiting to be flushed to - * disk. Basically this is the list of SyncItems whose callbacks will be - * invoked after flush returns successfully. - */ - LinkedList<Request> toFlush = new LinkedList<Request>(); - - FileOutputStream logStream; - - BinaryOutputArchive logArchive; - - Random r = new Random(System.nanoTime()); - - int logCount = 0; - - Request requestOfDeath = Request.requestOfDeath; - - private static ByteBuffer fill = ByteBuffer.allocateDirect(1024); - - LinkedList<FileOutputStream> streamsToFlush = new LinkedList<FileOutputStream>(); - - private long padLogFile(FileChannel fc,long fileSize) throws IOException{ - long position = fc.position(); - // We pad the file in 1M chunks to avoid syncing to - // write the new filesize. - if (position + 4096 >= fileSize) { - fileSize = fileSize + preAllocSize; - fill.position(0); - fc.write(fill, fileSize); - } - return fileSize; - } - - public void run() { - try { - long fileSize = 0; - long lastZxidSeen = -1; - FileChannel fc = null; - while (true) { - Request si = null; - if (toFlush.isEmpty()) { - si = queuedRequests.take(); - } else { - si = queuedRequests.poll(); - if (si == null) { - flush(toFlush); - continue; - } - } - if (si == requestOfDeath) { - break; - } - if (si != null) { - // LOG.warn("Sync>>> cxid = " + si.cxid + " type = " + - // si.type + " id = " + si.sessionId + " zxid = " + - // Long.toHexString(si.zxid)); - ZooTrace.logRequest(LOG, ZooTrace.CLIENT_REQUEST_TRACE_MASK, - 'S', si, ""); - TxnHeader hdr = si.hdr; - if (hdr != null) { - if (hdr.getZxid() <= lastZxidSeen) { - LOG.warn("Current zxid " + hdr.getZxid() - + " is <= " + lastZxidSeen + " for " - + hdr.getType()); - } - Record txn = si.txn; - if (logStream == null) { - fileSize = 0; - logStream = new FileOutputStream(new File( - zks.dataLogDir, ZooKeeperServer - .getLogName(hdr.getZxid()))); - synchronized (streamsToFlush) { - streamsToFlush.add(logStream); - } - fc = logStream.getChannel(); - logArchive = BinaryOutputArchive - .getArchive(logStream); - } - final long fsize=fileSize; - final FileChannel ffc=fc; - fileSize = Profiler.profile( - new Profiler.Operation<Long>() { - public Long execute() throws Exception { - return SyncRequestProcessor.this - .padLogFile(ffc, fsize); - } - }, PADDING_TIMEOUT, - "Logfile padding exceeded time threshold" - ); - ByteArrayOutputStream baos = new ByteArrayOutputStream(); - BinaryOutputArchive boa = BinaryOutputArchive - .getArchive(baos); - hdr.serialize(boa, "hdr"); - if (txn != null) { - txn.serialize(boa, "txn"); - } - logArchive.writeBuffer(baos.toByteArray(), "txnEntry"); - logArchive.writeByte((byte) 0x42, "EOR"); - logCount++; - if (logCount > snapCount / 2 - && r.nextInt(snapCount / 2) == 0) { - // We just want one snapshot going at a time - if (snapInProcess != null - && snapInProcess.isAlive()) { - LOG.warn("Too busy to snap, skipping"); - } else { - logStream = null; - logArchive = null; - snapInProcess = new Thread() { - public void run() { - try { - zks.snapshot(); - } catch (Exception e) { - LOG.warn("Unexpected exception",e); - } - } - }; - snapInProcess.start(); - } - logCount = 0; - } - } - toFlush.add(si); - if (toFlush.size() > 1000) { - flush(toFlush); - } - } - } - } catch (Exception e) { - LOG.error("Severe error, exiting",e); - System.exit(11); - } - ZooTrace.logTraceMessage(LOG, ZooTrace.getTextTraceLevel(), - "SyncRequestProcessor exiyed!"); - } - - private void flush(LinkedList<Request> toFlush) throws IOException { - if (toFlush.size() == 0) { - return; - } - - LinkedList<FileOutputStream> streamsToFlushNow; - synchronized (streamsToFlush) { - streamsToFlushNow = (LinkedList<FileOutputStream>) streamsToFlush - .clone(); - } - for (FileOutputStream fos : streamsToFlushNow) { - fos.flush(); - if (forceSync) { - ((FileChannel) fos.getChannel()).force(false); - } - } - while (streamsToFlushNow.size() > 1) { - FileOutputStream fos = streamsToFlushNow.removeFirst(); - fos.close(); - synchronized (streamsToFlush) { - streamsToFlush.remove(fos); - } - } - while (toFlush.size() > 0) { - Request i = toFlush.remove(); - nextProcessor.processRequest(i); - } - } - - public void shutdown() { - timeToDie = true; - queuedRequests.add(requestOfDeath); - nextProcessor.shutdown(); - } - - public void processRequest(Request request) { - // request.addRQRec(">sync"); - queuedRequests.add(request); - } - -} +/** + * 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. + */ + +package org.apache.zookeeper.server; + +import java.io.ByteArrayOutputStream; +import java.io.File; +import java.io.FileOutputStream; +import java.io.IOException; +import java.nio.ByteBuffer; +import java.nio.channels.FileChannel; +import java.util.LinkedList; +import java.util.Random; +import java.util.concurrent.LinkedBlockingQueue; + +import org.apache.log4j.Logger; + +import org.apache.jute.BinaryOutputArchive; +import org.apache.jute.Record; +import org.apache.zookeeper.server.util.Profiler; +import org.apache.zookeeper.txn.TxnHeader; + +/** + * This RequestProcessor logs requests to disk. It batches the requests to do + * the io efficiently. The request is not passed to the next RequestProcessor + * until its log has been synced to disk. + */ +public class SyncRequestProcessor extends Thread implements RequestProcessor { + private static final Logger LOG = Logger.getLogger(SyncRequestProcessor.class); + + static final int PADDING_TIMEOUT=1000; + ZooKeeperServer zks; + + LinkedBlockingQueue<Request> queuedRequests = new LinkedBlockingQueue<Request>(); + + static boolean forceSync; + static { + forceSync = !System.getProperty("zookeeper.forceSync", "yes").equals( + "no"); + } + + static long preAllocSize = 65536 * 1024; + static { + String size = System.getProperty("zookeeper.preAllocSize"); + if (size != null) { + try { + preAllocSize = Long.parseLong(size) * 1024; + } catch (NumberFormatException e) { + LOG.warn(size + " is not a valid value for preAllocSize"); + } + } + } + + /** + * The number of log entries to log before starting a snapshot + */ + static public int snapCount = ZooKeeperServer.getSnapCount(); + + Thread snapInProcess; + + RequestProcessor nextProcessor; + + boolean timeToDie = false; + + public SyncRequestProcessor(ZooKeeperServer zks, + RequestProcessor nextProcessor) { + super("SyncThread"); + this.zks = zks; + this.nextProcessor = nextProcessor; + start(); + } + + /** + * Transactions that have been written and are waiting to be flushed to + * disk. Basically this is the list of SyncItems whose callbacks will be + * invoked after flush returns successfully. + */ + LinkedList<Request> toFlush = new LinkedList<Request>(); + + FileOutputStream logStream; + + BinaryOutputArchive logArchive; + + Random r = new Random(System.nanoTime()); + + int logCount = 0; + + Request requestOfDeath = Request.requestOfDeath; + + private static ByteBuffer fill = ByteBuffer.allocateDirect(1024); + + LinkedList<FileOutputStream> streamsToFlush = new LinkedList<FileOutputStream>(); + + private long padLogFile(FileChannel fc,long fileSize) throws IOException{ + long position = fc.position(); + // We pad the file in 1M chunks to avoid syncing to + // write the new filesize. + if (position + 4096 >= fileSize) { + fileSize = fileSize + preAllocSize; + fill.position(0); + fc.write(fill, fileSize); + } + return fileSize; + } + + public void run() { + try { + long fileSize = 0; + long lastZxidSeen = -1; + FileChannel fc = null; + while (true) { + Request si = null; + if (toFlush.isEmpty()) { + si = queuedRequests.take(); + } else { + si = queuedRequests.poll(); + if (si == null) { + flush(toFlush); + continue; + } + } + if (si == requestOfDeath) { + break; + } + if (si != null) { + // LOG.warn("Sync>>> cxid = " + si.cxid + " type = " + + // si.type + " id = " + si.sessionId + " zxid = " + + // Long.toHexString(si.zxid)); + ZooTrace.logRequest(LOG, ZooTrace.CLIENT_REQUEST_TRACE_MASK, + 'S', si, ""); + TxnHeader hdr = si.hdr; + if (hdr != null) { + if (hdr.getZxid() <= lastZxidSeen) { + LOG.warn("Current zxid " + hdr.getZxid() + + " is <= " + lastZxidSeen + " for " + + hdr.getType()); + } + Record txn = si.txn; + if (logStream == null) { + fileSize = 0; + logStream = new FileOutputStream(new File( + zks.dataLogDir, ZooKeeperServer + .getLogName(hdr.getZxid()))); + synchronized (streamsToFlush) { + streamsToFlush.add(logStream); + } + fc = logStream.getChannel(); + logArchive = BinaryOutputArchive + .getArchive(logStream); + } + final long fsize=fileSize; + final FileChannel ffc=fc; + fileSize = Profiler.profile( + new Profiler.Operation<Long>() { + public Long execute() throws Exception { + return SyncRequestProcessor.this + .padLogFile(ffc, fsize); + } + }, PADDING_TIMEOUT, + "Logfile padding exceeded time threshold" + ); + ByteArrayOutputStream baos = new ByteArrayOutputStream(); + BinaryOutputArchive boa = BinaryOutputArchive + .getArchive(baos); + hdr.serialize(boa, "hdr"); + if (txn != null) { + txn.serialize(boa, "txn"); + } + logArchive.writeBuffer(baos.toByteArray(), "txnEntry"); + logArchive.writeByte((byte) 0x42, "EOR"); + logCount++; + if (logCount > snapCount / 2 + && r.nextInt(snapCount / 2) == 0) { + // We just want one snapshot going at a time + if (snapInProcess != null + && snapInProcess.isAlive()) { + LOG.warn("Too busy to snap, skipping"); + } else { + logStream = null; + logArchive = null; + snapInProcess = new Thread() { + public void run() { + try { + zks.snapshot(); + } catch (Exception e) { + LOG.warn("Unexpected exception",e); + } + } + }; + snapInProcess.start(); + } + logCount = 0; + } + } + toFlush.add(si); + if (toFlush.size() > 1000) { + flush(toFlush); + } + } + } + } catch (Exception e) { + LOG.error("Severe error, exiting",e); + System.exit(11); + } + ZooTrace.logTraceMessage(LOG, ZooTrace.getTextTraceLevel(), + "SyncRequestProcessor exiyed!"); + } + + private void flush(LinkedList<Request> toFlush) throws IOException { + if (toFlush.size() == 0) { + return; + } + + LinkedList<FileOutputStream> streamsToFlushNow; + synchronized (streamsToFlush) { + streamsToFlushNow = (LinkedList<FileOutputStream>) streamsToFlush + .clone(); + } + for (FileOutputStream fos : streamsToFlushNow) { + fos.flush(); + if (forceSync) { + ((FileChannel) fos.getChannel()).force(false); + } + } + while (streamsToFlushNow.size() > 1) { + FileOutputStream fos = streamsToFlushNow.removeFirst(); + fos.close(); + synchronized (streamsToFlush) { + streamsToFlush.remove(fos); + } + } + while (toFlush.size() > 0) { + Request i = toFlush.remove(); + nextProcessor.processRequest(i); + } + } + + public void shutdown() { + timeToDie = true; + queuedRequests.add(requestOfDeath); + nextProcessor.shutdown(); + } + + public void processRequest(Request request) { + // request.addRQRec(">sync"); + queuedRequests.add(request); + } + +}
Modified: hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/ZooTrace.java URL: http://svn.apache.org/viewvc/hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/ZooTrace.java?rev=671303&r1=671302&r2=671303&view=diff ============================================================================== --- hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/ZooTrace.java (original) +++ hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/ZooTrace.java Tue Jun 24 12:04:58 2008 @@ -1,92 +1,92 @@ -/** - * 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. - */ - -package org.apache.zookeeper.server; - -import org.apache.log4j.Logger; - -import org.apache.zookeeper.server.quorum.QuorumPacket; - -/** - * This class encapsulates and centralizes tracing for the ZooKeeper server. - * Trace messages go to the log with TRACE level. - * <p> - * Log4j must be correctly configured to capture the TRACE messages. - */ -public class ZooTrace { - final static public long CLIENT_REQUEST_TRACE_MASK = 1 << 1; - - final static public long CLIENT_DATA_PACKET_TRACE_MASK = 1 << 2; - - final static public long CLIENT_PING_TRACE_MASK = 1 << 3; - - final static public long SERVER_PACKET_TRACE_MASK = 1 << 4; - - final static public long SESSION_TRACE_MASK = 1 << 5; - - final static public long EVENT_DELIVERY_TRACE_MASK = 1 << 6; - - final static public long SERVER_PING_TRACE_MASK = 1 << 7; - - final static public long WARNING_TRACE_MASK = 1 << 8; - - final static public long JMX_TRACE_MASK = 1 << 9; - - private static long traceMask = CLIENT_REQUEST_TRACE_MASK - | SERVER_PACKET_TRACE_MASK | SESSION_TRACE_MASK - | WARNING_TRACE_MASK; - - public static long getTextTraceLevel() { - return traceMask; - } - - public static void setTextTraceLevel(long mask) { - traceMask = mask; - Logger LOG = Logger.getLogger(ZooTrace.class); - LOG.info("Set text trace mask to " + Long.toHexString(mask)); - } - - public static boolean isTraceEnabled(Logger log, long mask) { - return log.isTraceEnabled() && (mask & traceMask) != 0; - } - - public static void logTraceMessage(Logger log, long mask, String msg) { - if (isTraceEnabled(log, mask)) { - log.trace(msg); - } - } - - static public void logQuorumPacket(Logger log, long mask, - char direction, QuorumPacket qp) - { - return; - - // if (isTraceEnabled(log, mask)) { - // logTraceMessage(LOG, mask, direction + " " - // + FollowerHandler.packetToString(qp)); - // } - } - - static public void logRequest(Logger log, long mask, - char rp, Request request, String header) - { - if (isTraceEnabled(log, mask)) { - log.trace(header + ":" + rp + request.toString()); - } - } -} +/** + * 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. + */ + +package org.apache.zookeeper.server; + +import org.apache.log4j.Logger; + +import org.apache.zookeeper.server.quorum.QuorumPacket; + +/** + * This class encapsulates and centralizes tracing for the ZooKeeper server. + * Trace messages go to the log with TRACE level. + * <p> + * Log4j must be correctly configured to capture the TRACE messages. + */ +public class ZooTrace { + final static public long CLIENT_REQUEST_TRACE_MASK = 1 << 1; + + final static public long CLIENT_DATA_PACKET_TRACE_MASK = 1 << 2; + + final static public long CLIENT_PING_TRACE_MASK = 1 << 3; + + final static public long SERVER_PACKET_TRACE_MASK = 1 << 4; + + final static public long SESSION_TRACE_MASK = 1 << 5; + + final static public long EVENT_DELIVERY_TRACE_MASK = 1 << 6; + + final static public long SERVER_PING_TRACE_MASK = 1 << 7; + + final static public long WARNING_TRACE_MASK = 1 << 8; + + final static public long JMX_TRACE_MASK = 1 << 9; + + private static long traceMask = CLIENT_REQUEST_TRACE_MASK + | SERVER_PACKET_TRACE_MASK | SESSION_TRACE_MASK + | WARNING_TRACE_MASK; + + public static long getTextTraceLevel() { + return traceMask; + } + + public static void setTextTraceLevel(long mask) { + traceMask = mask; + Logger LOG = Logger.getLogger(ZooTrace.class); + LOG.info("Set text trace mask to " + Long.toHexString(mask)); + } + + public static boolean isTraceEnabled(Logger log, long mask) { + return log.isTraceEnabled() && (mask & traceMask) != 0; + } + + public static void logTraceMessage(Logger log, long mask, String msg) { + if (isTraceEnabled(log, mask)) { + log.trace(msg); + } + } + + static public void logQuorumPacket(Logger log, long mask, + char direction, QuorumPacket qp) + { + return; + + // if (isTraceEnabled(log, mask)) { + // logTraceMessage(LOG, mask, direction + " " + // + FollowerHandler.packetToString(qp)); + // } + } + + static public void logRequest(Logger log, long mask, + char rp, Request request, String header) + { + if (isTraceEnabled(log, mask)) { + log.trace(header + ":" + rp + request.toString()); + } + } +} Modified: hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/auth/DigestAuthenticationProvider.java URL: http://svn.apache.org/viewvc/hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/auth/DigestAuthenticationProvider.java?rev=671303&r1=671302&r2=671303&view=diff ============================================================================== --- hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/auth/DigestAuthenticationProvider.java (original) +++ hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/auth/DigestAuthenticationProvider.java Tue Jun 24 12:04:58 2008 @@ -1,127 +1,127 @@ -/** - * 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. - */ - -package org.apache.zookeeper.server.auth; - -import java.io.IOException; -import java.security.MessageDigest; -import java.security.NoSuchAlgorithmException; - -import org.apache.log4j.Logger; - -import org.apache.zookeeper.KeeperException; -import org.apache.zookeeper.data.Id; -import org.apache.zookeeper.server.ServerCnxn; - -public class DigestAuthenticationProvider implements AuthenticationProvider { - private static final Logger LOG = Logger.getLogger(DigestAuthenticationProvider.class); - - public final static String superDigest = "super:1wZ8qIvQBMTq0KPxMc6RQ/PCXKM="; - - public String getScheme() { - return "digest"; - } - - static final private String base64Encode(byte b[]) { - StringBuffer sb = new StringBuffer(); - for (int i = 0; i < b.length;) { - int pad = 0; - int v = (b[i++] & 0xff) << 16; - if (i < b.length) { - v |= (b[i++] & 0xff) << 8; - } else { - pad++; - } - if (i < b.length) { - v |= (b[i++] & 0xff); - } else { - pad++; - } - sb.append(encode(v >> 18)); - sb.append(encode(v >> 12)); - if (pad < 2) { - sb.append(encode(v >> 6)); - } else { - sb.append('='); - } - if (pad < 1) { - sb.append(encode(v)); - } else { - sb.append('='); - } - } - return sb.toString(); - } - - static final private char encode(int i) { - i &= 0x3f; - if (i < 26) { - return (char) ('A' + i); - } - if (i < 52) { - return (char) ('a' + i - 26); - } - if (i < 62) { - return (char) ('0' + i - 52); - } - return i == 62 ? '+' : '/'; - } - - static public String generateDigest(String idPassword) - throws NoSuchAlgorithmException { - String parts[] = idPassword.split(":", 2); - byte digest[] = MessageDigest.getInstance("SHA1").digest( - idPassword.getBytes()); - return parts[0] + ":" + base64Encode(digest); - } - - public int handleAuthentication(ServerCnxn cnxn, byte[] authData) { - String id = new String(authData); - try { - String digest = generateDigest(id); - if (digest.equals(superDigest)) { - cnxn.getAuthInfo().add(new Id("super", "")); - } - cnxn.getAuthInfo().add(new Id(getScheme(), digest)); - return KeeperException.Code.Ok; - } catch (NoSuchAlgorithmException e) { - LOG.error("Missing algorithm",e); - } - return KeeperException.Code.AuthFailed; - } - - public boolean isAuthenticated() { - return true; - } - - public boolean isValid(String id) { - String parts[] = id.split(":"); - return parts.length == 2; - } - - public boolean matches(String id, String aclExpr) { - return id.equals(aclExpr); - } - - public static void main(String args[]) throws IOException, - NoSuchAlgorithmException { - for (int i = 0; i < args.length; i++) { - System.out.println(args[i] + "->" + generateDigest(args[i])); - } - } -} +/** + * 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. + */ + +package org.apache.zookeeper.server.auth; + +import java.io.IOException; +import java.security.MessageDigest; +import java.security.NoSuchAlgorithmException; + +import org.apache.log4j.Logger; + +import org.apache.zookeeper.KeeperException; +import org.apache.zookeeper.data.Id; +import org.apache.zookeeper.server.ServerCnxn; + +public class DigestAuthenticationProvider implements AuthenticationProvider { + private static final Logger LOG = Logger.getLogger(DigestAuthenticationProvider.class); + + public final static String superDigest = "super:1wZ8qIvQBMTq0KPxMc6RQ/PCXKM="; + + public String getScheme() { + return "digest"; + } + + static final private String base64Encode(byte b[]) { + StringBuffer sb = new StringBuffer(); + for (int i = 0; i < b.length;) { + int pad = 0; + int v = (b[i++] & 0xff) << 16; + if (i < b.length) { + v |= (b[i++] & 0xff) << 8; + } else { + pad++; + } + if (i < b.length) { + v |= (b[i++] & 0xff); + } else { + pad++; + } + sb.append(encode(v >> 18)); + sb.append(encode(v >> 12)); + if (pad < 2) { + sb.append(encode(v >> 6)); + } else { + sb.append('='); + } + if (pad < 1) { + sb.append(encode(v)); + } else { + sb.append('='); + } + } + return sb.toString(); + } + + static final private char encode(int i) { + i &= 0x3f; + if (i < 26) { + return (char) ('A' + i); + } + if (i < 52) { + return (char) ('a' + i - 26); + } + if (i < 62) { + return (char) ('0' + i - 52); + } + return i == 62 ? '+' : '/'; + } + + static public String generateDigest(String idPassword) + throws NoSuchAlgorithmException { + String parts[] = idPassword.split(":", 2); + byte digest[] = MessageDigest.getInstance("SHA1").digest( + idPassword.getBytes()); + return parts[0] + ":" + base64Encode(digest); + } + + public int handleAuthentication(ServerCnxn cnxn, byte[] authData) { + String id = new String(authData); + try { + String digest = generateDigest(id); + if (digest.equals(superDigest)) { + cnxn.getAuthInfo().add(new Id("super", "")); + } + cnxn.getAuthInfo().add(new Id(getScheme(), digest)); + return KeeperException.Code.Ok; + } catch (NoSuchAlgorithmException e) { + LOG.error("Missing algorithm",e); + } + return KeeperException.Code.AuthFailed; + } + + public boolean isAuthenticated() { + return true; + } + + public boolean isValid(String id) { + String parts[] = id.split(":"); + return parts.length == 2; + } + + public boolean matches(String id, String aclExpr) { + return id.equals(aclExpr); + } + + public static void main(String args[]) throws IOException, + NoSuchAlgorithmException { + for (int i = 0; i < args.length; i++) { + System.out.println(args[i] + "->" + generateDigest(args[i])); + } + } +} Modified: hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/auth/ProviderRegistry.java URL: http://svn.apache.org/viewvc/hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/auth/ProviderRegistry.java?rev=671303&r1=671302&r2=671303&view=diff ============================================================================== --- hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/auth/ProviderRegistry.java (original) +++ hadoop/zookeeper/trunk/src/java/main/org/apache/zookeeper/server/auth/ProviderRegistry.java Tue Jun 24 12:04:58 2008 @@ -1,70 +1,70 @@ -/** - * 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. - */ - -package org.apache.zookeeper.server.auth; - -import java.util.Enumeration; -import java.util.HashMap; - -import org.apache.log4j.Logger; - -import org.apache.zookeeper.server.ZooKeeperServer; - -public class ProviderRegistry { - private static final Logger LOG = Logger.getLogger(ProviderRegistry.class); - - private static boolean initialized = false; - private static HashMap<String, AuthenticationProvider> authenticationProviders = - new HashMap<String, AuthenticationProvider>(); - - public static void initialize() { - synchronized (ProviderRegistry.class) { - if (initialized) - return; - initialized = true; - IPAuthenticationProvider ipp = new IPAuthenticationProvider(); - HostAuthenticationProvider hostp = new HostAuthenticationProvider(); - DigestAuthenticationProvider digp = new DigestAuthenticationProvider(); - authenticationProviders.put(ipp.getScheme(), ipp); - authenticationProviders.put(hostp.getScheme(), hostp); - authenticationProviders.put(digp.getScheme(), digp); - Enumeration<Object> en = System.getProperties().keys(); - while (en.hasMoreElements()) { - String k = (String) en.nextElement(); - if (k.startsWith("zookeeper.authProvider.")) { - String className = System.getProperty(k); - try { - Class<?> c = ZooKeeperServer.class.getClassLoader() - .loadClass(className); - AuthenticationProvider ap = (AuthenticationProvider) c - .newInstance(); - authenticationProviders.put(ap.getScheme(), ap); - } catch (Exception e) { - LOG.warn("Problems loading " + className,e); - } - } - } - } - } - - public static AuthenticationProvider getProvider(String scheme) { - if(!initialized) - initialize(); - return authenticationProviders.get(scheme); - } -} +/** + * 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. + */ + +package org.apache.zookeeper.server.auth; + +import java.util.Enumeration; +import java.util.HashMap; + +import org.apache.log4j.Logger; + +import org.apache.zookeeper.server.ZooKeeperServer; + +public class ProviderRegistry { + private static final Logger LOG = Logger.getLogger(ProviderRegistry.class); + + private static boolean initialized = false; + private static HashMap<String, AuthenticationProvider> authenticationProviders = + new HashMap<String, AuthenticationProvider>(); + + public static void initialize() { + synchronized (ProviderRegistry.class) { + if (initialized) + return; + initialized = true; + IPAuthenticationProvider ipp = new IPAuthenticationProvider(); + HostAuthenticationProvider hostp = new HostAuthenticationProvider(); + DigestAuthenticationProvider digp = new DigestAuthenticationProvider(); + authenticationProviders.put(ipp.getScheme(), ipp); + authenticationProviders.put(hostp.getScheme(), hostp); + authenticationProviders.put(digp.getScheme(), digp); + Enumeration<Object> en = System.getProperties().keys(); + while (en.hasMoreElements()) { + String k = (String) en.nextElement(); + if (k.startsWith("zookeeper.authProvider.")) { + String className = System.getProperty(k); + try { + Class<?> c = ZooKeeperServer.class.getClassLoader() + .loadClass(className); + AuthenticationProvider ap = (AuthenticationProvider) c + .newInstance(); + authenticationProviders.put(ap.getScheme(), ap); + } catch (Exception e) { + LOG.warn("Problems loading " + className,e); + } + } + } + } + } + + public static AuthenticationProvider getProvider(String scheme) { + if(!initialized) + initialize(); + return authenticationProviders.get(scheme); + } +}