Return-Path: X-Original-To: apmail-zookeeper-commits-archive@www.apache.org Delivered-To: apmail-zookeeper-commits-archive@www.apache.org Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by minotaur.apache.org (Postfix) with SMTP id 2034118357 for ; Thu, 3 Dec 2015 04:30:30 +0000 (UTC) Received: (qmail 13766 invoked by uid 500); 3 Dec 2015 04:30:30 -0000 Delivered-To: apmail-zookeeper-commits-archive@zookeeper.apache.org Received: (qmail 13739 invoked by uid 500); 3 Dec 2015 04:30:30 -0000 Mailing-List: contact commits-help@zookeeper.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@ Delivered-To: mailing list commits@zookeeper.apache.org Received: (qmail 13728 invoked by uid 99); 3 Dec 2015 04:30:29 -0000 Received: from Unknown (HELO spamd2-us-west.apache.org) (209.188.14.142) by apache.org (qpsmtpd/0.29) with ESMTP; Thu, 03 Dec 2015 04:30:29 +0000 Received: from localhost (localhost [127.0.0.1]) by spamd2-us-west.apache.org (ASF Mail Server at spamd2-us-west.apache.org) with ESMTP id 5343B1A22E1 for ; Thu, 3 Dec 2015 04:30:29 +0000 (UTC) X-Virus-Scanned: Debian amavisd-new at spamd2-us-west.apache.org X-Spam-Flag: NO X-Spam-Score: 1.247 X-Spam-Level: * X-Spam-Status: No, score=1.247 tagged_above=-999 required=6.31 tests=[KAM_ASCII_DIVIDERS=0.8, KAM_LAZY_DOMAIN_SECURITY=1, RP_MATCHES_RCVD=-0.554, URIBL_BLOCKED=0.001] autolearn=disabled Received: from mx1-us-west.apache.org ([10.40.0.8]) by localhost (spamd2-us-west.apache.org [10.40.0.9]) (amavisd-new, port 10024) with ESMTP id GmvQn9_JGGC7 for ; Thu, 3 Dec 2015 04:30:09 +0000 (UTC) Received: from mailrelay1-us-west.apache.org (mailrelay1-us-west.apache.org [209.188.14.139]) by mx1-us-west.apache.org (ASF Mail Server at mx1-us-west.apache.org) with ESMTP id EAF1F35C43 for ; Thu, 3 Dec 2015 04:29:49 +0000 (UTC) Received: from svn01-us-west.apache.org (svn.apache.org [10.41.0.6]) by mailrelay1-us-west.apache.org (ASF Mail Server at mailrelay1-us-west.apache.org) with ESMTP id AB305E10A7 for ; Thu, 3 Dec 2015 04:29:48 +0000 (UTC) Received: from svn01-us-west.apache.org (localhost [127.0.0.1]) by svn01-us-west.apache.org (ASF Mail Server at svn01-us-west.apache.org) with ESMTP id A33383A0D1C for ; Thu, 3 Dec 2015 04:29:48 +0000 (UTC) Content-Type: text/plain; charset="utf-8" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit Subject: svn commit: r1717709 [28/43] - in /zookeeper/site/trunk: content/ content/doc/r3.4.7/ content/doc/r3.4.7/api/ content/doc/r3.4.7/api/org/ content/doc/r3.4.7/api/org/apache/ content/doc/r3.4.7/api/org/apache/zookeeper/ content/doc/r3.4.7/api/org/apache/... Date: Thu, 03 Dec 2015 04:29:45 -0000 To: commits@zookeeper.apache.org From: rgs@apache.org X-Mailer: svnmailer-1.0.9 Message-Id: <20151203042948.A33383A0D1C@svn01-us-west.apache.org> Added: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperOverview.html URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperOverview.html?rev=1717709&view=auto ============================================================================== --- zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperOverview.html (added) +++ zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperOverview.html Thu Dec 3 04:29:41 2015 @@ -0,0 +1,694 @@ + + + + + + + +BookKeeper overview + + + + + + + + + +
+ + + +
+ + + + + + + + + + + + +
+
+
+
+ +
+ + +
+ +
+ +   +
+ + + + + +
+ +

BookKeeper overview

+ + + + + + +

BookKeeper overview

+
+ +

BookKeeper introduction

+

+ BookKeeper is a replicated service to reliably log streams of records. In BookKeeper, + servers are "bookies", log streams are "ledgers", and each unit of a log (aka record) is a + "ledger entry". BookKeeper is designed to be reliable; bookies, the servers that store + ledgers, can crash, corrupt data, discard data, but as long as there are enough bookies + behaving correctly the service as a whole behaves correctly. +

+

+ The initial motivation for BookKeeper comes from the namenode of HDFS. Namenodes have to + log operations in a reliable fashion so that recovery is possible in the case of crashes. + We have found the applications for BookKeeper extend far beyond HDFS, however. Essentially, + any application that requires an append storage can replace their implementations with + BookKeeper. BookKeeper has the advantage of scaling throughput with the number of servers. +

+

+ At a high level, a bookkeeper client receives entries from a client application and stores it to + sets of bookies, and there are a few advantages in having such a service: +

+
    + +
  • + +

    + We can use hardware that is optimized for such a service. We currently believe that such a + system has to be optimized only for disk I/O; +

    + +
  • + + +
  • + +

    + We can have a pool of servers implementing such a log system, and shared among a number of servers; +

    + +
  • + + +
  • + +

    + We can have a higher degree of replication with such a pool, which makes sense if the hardware necessary for it is cheaper compared to the one the application uses. +

    + +
  • + +
+ +

In slightly more detail...

+

BookKeeper implements highly available logs, and it has been designed with write-ahead logging in mind. Besides high availability + due to the replicated nature of the service, it provides high throughput due to striping. As we write entries in a subset of bookies of an + ensemble and rotate writes across available quorums, we are able to increase throughput with the number of servers for both reads and writes. + Scalability is a property that is possible to achieve in this case due to the use of quorums. Other replication techniques, such as + state-machine replication, do not enable such a property. +

+

An application first creates a ledger before writing to bookies through a local BookKeeper client instance. + Upon creating a ledger, a BookKeeper client writes metadata about the ledger to ZooKeeper. Each ledger currently + has a single writer. This writer has to execute a close ledger operation before any other client can read from it. + If the writer of a ledger does not close a ledger properly because, for example, it has crashed before having the + opportunity of closing the ledger, then the next client that tries to open a ledger executes a procedure to recover + it. As closing a ledger consists essentially of writing the last entry written to a ledger to ZooKeeper, the recovery + procedure simply finds the last entry written correctly and writes it to ZooKeeper. +

+

+ Note that currently this recovery procedure is executed automatically upon trying to open a ledger and no explicit action is necessary. + Although two clients may try to recover a ledger concurrently, only one will succeed, the first one that is able to create the close znode + for the ledger. +

+ +

Bookkeeper elements and concepts

+

+ BookKeeper uses four basic elements: +

+
    + +
  • + +

    + +Ledger: A ledger is a sequence of entries, and each entry is a sequence of bytes. Entries are + written sequentially to a ledger and at most once. Consequently, ledgers have an append-only semantics; +

    + +
  • + + +
  • + +

    + +BookKeeper client: A client runs along with a BookKeeper application, and it enables applications + to execute operations on ledgers, such as creating a ledger and writing to it; +

    + +
  • + + +
  • + +

    + +Bookie: A bookie is a BookKeeper storage server. Bookies store the content of ledgers. For any given + ledger L, we call an ensemble the group of bookies storing the content of L. For performance, we store on + each bookie of an ensemble only a fragment of a ledger. That is, we stripe when writing entries to a ledger such that + each entry is written to sub-group of bookies of the ensemble. +

    + +
  • + + +
  • + +

    + +Metadata storage service: BookKeeper requires a metadata storage service to store information related + to ledgers and available bookies. We currently use ZooKeeper for such a task. +

    + +
  • + +
+ +

Bookkeeper initial design

+

+ A set of bookies implements BookKeeper, and we use a quorum-based protocol to replicate data across the bookies. + There are basically two operations to an existing ledger: read and append. Here is the complete API list + (mode detail + here): +

+
    + +
  • + +

    + Create ledger: creates a new empty ledger; +

    + +
  • + + +
  • + +

    + Open ledger: opens an existing ledger for reading; +

    + +
  • + + +
  • + +

    + Add entry: adds a record to a ledger either synchronously or asynchronously; +

    + +
  • + + +
  • + +

    + Read entries: reads a sequence of entries from a ledger either synchronously or asynchronously +

    + +
  • + +
+

+ There is only a single client that can write to a ledger. Once that ledger is closed or the client fails, + no more entries can be added. (We take advantage of this behavior to provide our strong guarantees.) + There will not be gaps in the ledger. Fingers get broken, people get roughed up or end up in prison when + books are manipulated, so there is no deleting or changing of entries. +

+ + + + + + + +
BookKeeper Overview
+ + + +
+

+ A simple use of BooKeeper is to implement a write-ahead transaction log. A server maintains an in-memory data structure + (with periodic snapshots for example) and logs changes to that structure before it applies the change. The application + server creates a ledger at startup and store the ledger id and password in a well known place (ZooKeeper maybe). When + it needs to make a change, the server adds an entry with the change information to a ledger and apply the change when + BookKeeper adds the entry successfully. The server can even use asyncAddEntry to queue up many changes for high change + throughput. BooKeeper meticulously logs the changes in order and call the completion functions in order. +

+

+ When the application server dies, a backup server will come online, get the last snapshot and then it will open the + ledger of the old server and read all the entries from the time the snapshot was taken. (Since it doesn't know the + last entry number it will use MAX_INTEGER). Once all the entries have been processed, it will close the ledger and + start a new one for its use. +

+

+ A client library takes care of communicating with bookies and managing entry numbers. An entry has the following fields: +

+ + +Entry fields + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + + +
Entry fields
FieldTypeDescription
Ledger numberlongThe id of the ledger of this entry
Entry numberlongThe id of this entry
last confirmed (LC)longid of the last recorded entry
databyte[]the entry data (supplied by application)
authentication codebyte[]Message authentication code that includes all other fields of the entry
+

+ The client library generates a ledger entry. None of the fields are modified by the bookies and only the first three + fields are interpreted by the bookies. +

+

+ To add to a ledger, the client generates the entry above using the ledger number. The entry number will be one more + than the last entry generated. The LC field contains the last entry that has been successfully recorded by BookKeeper. + If the client writes entries one at a time, LC is the last entry id. But, if the client is using asyncAddEntry, there + may be many entries in flight. An entry is considered recorded when both of the following conditions are met: +

+
    + +
  • + +

    + the entry has been accepted by a quorum of bookies +

    + +
  • + + +
  • + +

    + all entries with a lower entry id have been accepted by a quorum of bookies +

    + +
  • + +
+

+ +LC seems mysterious right now, but it is too early to explain how we use it; just smile and move on. +

+

+ Once all the other fields have been field in, the client generates an authentication code with all of the previous fields. + The entry is then sent to a quorum of bookies to be recorded. Any failures will result in the entry being sent to a new + quorum of bookies. +

+

+ To read, the client library initially contacts a bookie and starts requesting entries. If an entry is missing or + invalid (a bad MAC for example), the client will make a request to a different bookie. By using quorum writes, + as long as enough bookies are up we are guaranteed to eventually be able to read an entry. +

+ +

Bookkeeper metadata management

+

+ There are some meta data that needs to be made available to BookKeeper clients: +

+
    + +
  • + +

    + The available bookies; +

    + +
  • + + +
  • + +

    + The list of ledgers; +

    + +
  • + + +
  • + +

    + The list of bookies that have been used for a given ledger; +

    + +
  • + + +
  • + +

    + The last entry of a ledger; +

    + +
  • + +
+

+ We maintain this information in ZooKeeper. Bookies use ephemeral nodes to indicate their availability. Clients + use znodes to track ledger creation and deletion and also to know the end of the ledger and the bookies that + were used to store the ledger. Bookies also watch the ledger list so that they can cleanup ledgers that get deleted. +

+ +

Closing out ledgers

+

+ The process of closing out the ledger and finding the last ledger is difficult due to the durability guarantees of BookKeeper: +

+
    + +
  • + +

    + If an entry has been successfully recorded, it must be readable. +

    + +
  • + + +
  • + +

    + If an entry is read once, it must always be available to be read. +

    + +
  • + +
+

+ If the ledger was closed gracefully, ZooKeeper will have the last entry and everything will work well. But, if the + BookKeeper client that was writing the ledger dies, there is some recovery that needs to take place. +

+

+ The problematic entries are the ones at the end of the ledger. There can be entries in flight when a BookKeeper client + dies. If the entry only gets to one bookie, the entry should not be readable since the entry will disappear if that bookie + fails. If the entry is only on one bookie, that doesn't mean that the entry has not been recorded successfully; the other + bookies that recorded the entry might have failed. +

+

+ The trick to making everything work is to have a correct idea of a last entry. We do it in roughly three steps: +

+
    + +
  1. + +

    + Find the entry with the highest last recorded entry, LC; +

    + +
  2. + + +
  3. + +

    + Find the highest consecutively recorded entry, LR; +

    + +
  4. + + +
  5. + +

    + Make sure that all entries between LC and LR are on a quorum of bookies; +

    + +
  6. + + +
+
+ +

+ +

+
+ +
 
+
+ + + Added: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperOverview.pdf URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperOverview.pdf?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperOverview.pdf ------------------------------------------------------------------------------ svn:mime-type = application/pdf Added: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperProgrammer.html URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperProgrammer.html?rev=1717709&view=auto ============================================================================== --- zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperProgrammer.html (added) +++ zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperProgrammer.html Thu Dec 3 04:29:41 2015 @@ -0,0 +1,1083 @@ + + + + + + + +BookKeeper Getting Started Guide + + + + + + + + + +
+ + + +
+ + + + + + + + + + + + +
+
+
+
+ +
+ + +
+ +
+ +   +
+ + + + + +
+ +

BookKeeper Getting Started Guide

+ + + + + + +

Programming with BookKeeper

+
+ + +

Instantiating BookKeeper.

+

+ The first step to use BookKeeper is to instantiate a BookKeeper object: +

+

+ + + org.apache.bookkeeper.BookKeeper + + +

+

+ There are three BookKeeper constructors: +

+

+ + + public BookKeeper(String servers) + throws KeeperException, IOException + + +

+

+ where: +

+
    + +
  • + +

    + +servers is a comma-separated list of ZooKeeper servers. +

    + +
  • + +
+

+ + + public BookKeeper(ZooKeeper zk) + throws InterruptedException, KeeperException + + +

+

+ where: +

+
    + +
  • + +

    + +zk is a ZooKeeper object. This constructor is useful when + the application also using ZooKeeper and wants to have a single instance of ZooKeeper. +

    + +
  • + +
+

+ + + public BookKeeper(ZooKeeper zk, ClientSocketChannelFactory channelFactory) + throws InterruptedException, KeeperException + + +

+

+ where: +

+
    + +
  • + +

    + +zk is a ZooKeeper object. This constructor is useful when + the application also using ZooKeeper and wants to have a single instance of ZooKeeper. +

    + +
  • + + +
  • + +

    + +channelFactory is a netty channel object + (org.jboss.netty.channel.socket). +

    + +
  • + +
+ +

Creating a ledger.

+

Before writing entries to BookKeeper, it is necessary to create a ledger. + With the current BookKeeper API, it is possible to create a ledger both synchronously + or asynchronously. The following methods belong + to org.apache.bookkeeper.client.BookKeeper. +

+

+ +Synchronous call: + +

+

+ + + public LedgerHandle createLedger(int ensSize, int qSize, DigestType type, byte passwd[]) + throws KeeperException, InterruptedException, + IOException, BKException + + +

+

+ where: +

+
    + +
  • + +

    + +ensSize is the number of bookies (ensemble size); +

    + +
  • + + +
  • + +

    + +qSize is the write quorum size; +

    + +
  • + + +
  • + +

    + +type is the type of digest used with entries: either MAC or CRC32. +

    + +
  • + + +
  • + +

    + +passwd is a password that authorizes the client to write to the + ledger being created. +

    + +
  • + +
+

+ All further operations on a ledger are invoked through the LedgerHandle + object returned. +

+

+ As a convenience, we provide a createLedger with default parameters (3,2,VERIFIABLE), + and the only two input parameters it requires are a digest type and a password. +

+

+ +Asynchronous call: + +

+

+ + + public void asyncCreateLedger(int ensSize, + int qSize, + DigestType type, + byte passwd[], + CreateCallback cb, + Object ctx + ) + + +

+

+ The parameters are the same of the synchronous version, with the + exception of cb and ctx. CreateCallback + is an interface in org.apache.bookkeeper.client.AsyncCallback, and + a class implementing it has to implement a method called createComplete + that has the following signature: +

+

+ + + void createComplete(int rc, LedgerHandle lh, Object ctx); + + +

+

+ where: +

+
    + +
  • + +

    + +rc is a return code (please refer to org.apache.bookeeper.client.BKException for a list); +

    + +
  • + + +
  • + +

    + +lh is a LedgerHandle object to manipulate a ledger; +

    + +
  • + + +
  • + +

    + +ctx is a control object for accountability purposes. It can be essentially any object the application is happy with. +

    + +
  • + +
+

+ The ctx object passed as a parameter to the call to create a ledger + is the one same returned in the callback. +

+ +

Adding entries to a ledger.

+

+ Once we have a ledger handle lh obtained through a call to create a ledger, we + can start writing entries. As with creating ledgers, we can write both synchronously and + asynchronously. The following methods belong + to org.apache.bookkeeper.client.LedgerHandle. +

+

+ +Synchronous call: + +

+

+ + + public long addEntry(byte[] data) + throws InterruptedException + + +

+

+ where: +

+
    + +
  • + +

    + +data is a byte array; +

    + +
  • + +
+

+ A call to addEntry returns the status of the operation (please refer to org.apache.bookeeper.client.BKDefs for a list); +

+

+ +Asynchronous call: + +

+

+ + + public void asyncAddEntry(byte[] data, AddCallback cb, Object ctx) + + +

+

+ It also takes a byte array as the sequence of bytes to be stored as an entry. Additionaly, it takes + a callback object cb and a control object ctx. The callback object must implement + the AddCallback interface in org.apache.bookkeeper.client.AsyncCallback, and + a class implementing it has to implement a method called addComplete + that has the following signature: +

+

+ + + void addComplete(int rc, LedgerHandle lh, long entryId, Object ctx); + + +

+

+ where: +

+
    + +
  • + +

    + +rc is a return code (please refer to org.apache.bookeeper.client.BKDefs for a list); +

    + +
  • + + +
  • + +

    + +lh is a LedgerHandle object to manipulate a ledger; +

    + +
  • + + +
  • + +

    + +entryId is the identifier of entry associated with this request; +

    + +
  • + + +
  • + +

    + +ctx is control object used for accountability purposes. It can be any object the application is happy with. +

    + +
  • + +
+ +

Closing a ledger.

+

+ Once a client is done writing, it closes the ledger. The following methods belong + to org.apache.bookkeeper.client.LedgerHandle. +

+

+ +Synchronous close: + +

+

+ + + public void close() + throws InterruptedException + + +

+

+ It takes no input parameters. +

+

+ +Asynchronous close: + +

+

+ + + public void asyncClose(CloseCallback cb, Object ctx) + throws InterruptedException + + +

+

+ It takes a callback object cb and a control object ctx. The callback object must implement + the CloseCallback interface in org.apache.bookkeeper.client.AsyncCallback, and + a class implementing it has to implement a method called closeComplete + that has the following signature: +

+

+ + + void closeComplete(int rc, LedgerHandle lh, Object ctx) + + +

+

+ where: +

+
    + +
  • + +

    + +rc is a return code (please refer to org.apache.bookeeper.client.BKDefs for a list); +

    + +
  • + + +
  • + +

    + +lh is a LedgerHandle object to manipulate a ledger; +

    + +
  • + + +
  • + +

    + +ctx is control object used for accountability purposes. +

    + +
  • + +
+ +

Opening a ledger.

+

+ To read from a ledger, a client must open it first. The following methods belong + to org.apache.bookkeeper.client.BookKeeper. +

+

+ +Synchronous open: + +

+

+ + + public LedgerHandle openLedger(long lId, DigestType type, byte passwd[]) + throws InterruptedException, BKException + + +

+
    + +
  • + +

    + +ledgerId is the ledger identifier; +

    + +
  • + + +
  • + +

    + +type is the type of digest used with entries: either MAC or CRC32. +

    + +
  • + + +
  • + +

    + +passwd is a password to access the ledger (used only in the case of VERIFIABLE ledgers); +

    + +
  • + +
+

+ +Asynchronous open: + +

+

+ + + public void asyncOpenLedger(long lId, DigestType type, byte passwd[], OpenCallback cb, Object ctx) + + +

+

+ It also takes a a ledger identifier and a password. Additionaly, it takes a callback object + cb and a control object ctx. The callback object must implement + the OpenCallback interface in org.apache.bookkeeper.client.AsyncCallback, and + a class implementing it has to implement a method called openComplete + that has the following signature: +

+

+ + + public void openComplete(int rc, LedgerHandle lh, Object ctx) + + +

+

+ where: +

+
    + +
  • + +

    + +rc is a return code (please refer to org.apache.bookeeper.client.BKDefs for a list); +

    + +
  • + + +
  • + +

    + +lh is a LedgerHandle object to manipulate a ledger; +

    + +
  • + + +
  • + +

    + +ctx is control object used for accountability purposes. +

    + +
  • + +
+ +

Reading from ledger

+

+ Read calls may request one or more consecutive entries. The following methods belong + to org.apache.bookkeeper.client.LedgerHandle. +

+

+ +Synchronous read: + +

+

+ + + public Enumeration<LedgerEntry> readEntries(long firstEntry, long lastEntry) + throws InterruptedException, BKException + + +

+
    + +
  • + +

    + +firstEntry is the identifier of the first entry in the sequence of entries to read; +

    + +
  • + + +
  • + +

    + +lastEntry is the identifier of the last entry in the sequence of entries to read. +

    + +
  • + +
+

+ +Asynchronous read: + +

+

+ + + public void asyncReadEntries(long firstEntry, + long lastEntry, ReadCallback cb, Object ctx) + throws BKException, InterruptedException + + +

+

+ It also takes a first and a last entry identifiers. Additionaly, it takes a callback object + cb and a control object ctx. The callback object must implement + the ReadCallback interface in org.apache.bookkeeper.client.AsyncCallback, and + a class implementing it has to implement a method called readComplete + that has the following signature: +

+

+ + + void readComplete(int rc, LedgerHandle lh, Enumeration<LedgerEntry> seq, Object ctx) + + +

+

+ where: +

+
    + +
  • + +

    + +rc is a return code (please refer to org.apache.bookeeper.client.BKDefs for a list); +

    + +
  • + + +
  • + +

    + +lh is a LedgerHandle object to manipulate a ledger; +

    + +
  • + + +
  • + +

    + +seq is a Enumeration<LedgerEntry> object to containing the list of entries requested; +

    + +
  • + + +
  • + +

    + +ctx is control object used for accountability purposes. +

    + +
  • + +
+ +

Deleting a ledger

+

+ Once a client is done with a ledger and is sure that nobody will ever need to read from it again, they can delete the ledger. + The following methods belong to org.apache.bookkeeper.client.BookKeeper. +

+

+ +Synchronous delete: + +

+

+ + + public void deleteLedger(long lId) throws InterruptedException, BKException + + +

+
    + +
  • + +

    + +lId is the ledger identifier; +

    + +
  • + +
+

+ +Asynchronous delete: + +

+

+ + + public void asyncDeleteLedger(long lId, DeleteCallback cb, Object ctx) + + +

+

+ It takes a ledger identifier. Additionally, it takes a callback object + cb and a control object ctx. The callback object must implement + the DeleteCallback interface in org.apache.bookkeeper.client.AsyncCallback, and + a class implementing it has to implement a method called deleteComplete + that has the following signature: +

+

+ + + void deleteComplete(int rc, Object ctx) + + +

+

+ where: +

+
    + +
  • + +

    + +rc is a return code (please refer to org.apache.bookeeper.client.BKDefs for a list); +

    + +
  • + + +
  • + +

    + +ctx is control object used for accountability purposes. +

    + +
  • + +
+
+ +

+ +

+
+ +
 
+
+ + + Added: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperProgrammer.pdf URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperProgrammer.pdf?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperProgrammer.pdf ------------------------------------------------------------------------------ svn:mime-type = application/pdf Added: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStarted.html URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStarted.html?rev=1717709&view=auto ============================================================================== --- zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStarted.html (added) +++ zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStarted.html Thu Dec 3 04:29:41 2015 @@ -0,0 +1,448 @@ + + + + + + + +BookKeeper Getting Started Guide + + + + + + + + + +
+ + + +
+ + + + + + + + + + + + +
+
+
+
+ +
+ + +
+ +
+ +   +
+ + + + + +
+ +

BookKeeper Getting Started Guide

+ + + + + + +

Getting Started: Setting up BookKeeper to write logs.

+
+

This document contains information to get you started quickly with + BookKeeper. It is aimed primarily at developers willing to try it out, and + contains simple installation instructions for a simple BookKeeper installation + and a simple programming example. For further programming detail, please refer to + BookKeeper Programmer's Guide. +

+ +

Pre-requisites

+

See + System Requirements in the Admin guide.

+ +

Download

+

BookKeeper is distributed along with ZooKeeper. To get a ZooKeeper distribution, + download a recent + + stable release from one of the Apache Download + Mirrors.

+ +

LocalBookKeeper

+

Under org.apache.bookkeeper.util, you'll find a java program + called LocalBookKeeper.java that sets you up to run BookKeeper on a + single machine. This is far from ideal from a performance perspective, + but the program is useful for both test and educational purposes. +

+ +

Setting up bookies

+

If you're bold and you want more than just running things locally, then + you'll need to run bookies in different servers. You'll need at least three bookies + to start with. +

+

+ For each bookie, we need to execute a command like the following: +

+

+ + java -cp .:./zookeeper-<version>-bookkeeper.jar:./zookeeper-<version>.jar\ + :lib/slf4j-api-1.6.1.jar:lib/slf4j-log4j12-1.6.1.jar:lib/log4j-1.2.15.jar -Dlog4j.configuration=log4j.properties\ + org.apache.bookkeeper.proto.BookieServer 3181 127.0.0.1:2181 /path_to_log_device/\ + /path_to_ledger_device/ + +

+

"/path_to_log_device/" and "/path_to_ledger_device/" are different paths. Also, port 3181 + is the port that a bookie listens on for connection requests from clients. 127.0.0.1:2181 is the hostname:port + for the ZooKeeper server. In this example, the standalone ZooKeeper server is running locally on port 2181. + If we had multiple ZooKeeper servers, this parameter would be a comma separated list of all the hostname:port + values corresponding to them. +

+ +

Setting up ZooKeeper

+

ZooKeeper stores metadata on behalf of BookKeeper clients and bookies. To get a minimal + ZooKeeper installation to work with BookKeeper, we can set up one server running in + standalone mode. Once we have the server running, we need to create a few znodes: +

+
    + +
  1. + +

    + + /ledgers + +

    + +
  2. + + +
  3. + +

    + + /ledgers/available + +

    + +
  4. + + +
  5. + +

    For each bookie, we add one znode such that the name of the znode is the + concatenation of the machine name and the port number that the bookie is + listening on. For example, if a bookie is running on bookie.foo.com an is listening + on port 3181, we add a znode + /ledgers/available/bookie.foo.com:3181. +

    + +
  6. + +
+ +

Example

+

+ In the following excerpt of code, we: +

+
    + +
  1. + +

    + Create a ledger; +

    + +
  2. + + +
  3. + +

    + Write to the ledger; +

    + +
  4. + + +
  5. + +

    + Close the ledger; +

    + +
  6. + + +
  7. + +

    + Open the same ledger for reading; +

    + +
  8. + + +
  9. + +

    + Read from the ledger; +

    + +
  10. + + +
  11. + +

    + Close the ledger again; +

    + +
  12. + +
+
+LedgerHandle lh = bkc.createLedger(ledgerPassword);
+ledgerId = lh.getId();
+ByteBuffer entry = ByteBuffer.allocate(4);
+
+for(int i = 0; i < 10; i++){
+	entry.putInt(i);
+	entry.position(0);
+	entries.add(entry.array());				
+	lh.addEntry(entry.array());
+}
+lh.close();
+lh = bkc.openLedger(ledgerId, ledgerPassword);		
+			
+Enumeration<LedgerEntry> ls = lh.readEntries(0, 9);
+int i = 0;
+while(ls.hasMoreElements()){
+	ByteBuffer origbb = ByteBuffer.wrap(
+				entries.get(i++));
+	Integer origEntry = origbb.getInt();
+	ByteBuffer result = ByteBuffer.wrap(
+				ls.nextElement().getEntry());
+
+	Integer retrEntry = result.getInt();
+}
+lh.close();
+	    
+
+ +

+ +

+
+ +
 
+
+ + + Added: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStarted.pdf URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStarted.pdf?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStarted.pdf ------------------------------------------------------------------------------ svn:mime-type = application/pdf Added: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStream.html URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStream.html?rev=1717709&view=auto ============================================================================== --- zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStream.html (added) +++ zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStream.html Thu Dec 3 04:29:41 2015 @@ -0,0 +1,612 @@ + + + + + + + +Streaming with BookKeeper + + + + + + + + + +
+ + + +
+ + + + + + + + + + + + +
+
+
+
+ +
+ + +
+ +
+ +   +
+ + + + + +
+ +

Streaming with BookKeeper

+ + + + + + +

Summary

+
+

+ When using the BookKeeper API, an application has to split the data to write into entries, each + entry being a byte array. This is natural for many applications. For example, when using BookKeeper + for write-ahead logging, an application typically wants to write the modifications corresponding + to a command or a transaction. Some other applications, however, might not have a natural boundary + for entries, and may prefer to write and read streams of bytes. This is exactly the purpose of the + stream API we have implemented on top of BookKeeper. +

+

+ The stream API is implemented in the package Streaming, and it contains two main classes: LedgerOutputStream and + LedgerInputStream. The class names are indicative of what they do. +

+
+ + + +

Writing a stream of bytes

+
+

+ Class LedgerOutputStream implements two constructors and five public methods: +

+

+ + + public LedgerOutputStream(LedgerHandle lh) + + +

+

+ where: +

+
    + +
  • + +

    + +lh is a ledger handle for a previously created and open ledger. +

    + +
  • + +
+

+ + + public LedgerOutputStream(LedgerHandle lh, int size) + + +

+

+ where: +

+
    + +
  • + +

    + +lh is a ledger handle for a previously created and open ledger. +

    + +
  • + + +
  • + +

    + +size is the size of the byte buffer to store written bytes before flushing. +

    + +
  • + +
+

+ +Closing a stream. This call closes the stream by flushing the write buffer. +

+

+ + + public void close() + + +

+

+ which has no parameters. +

+

+ +Flushing a stream. This call essentially flushes the write buffer. +

+

+ + + public synchronized void flush() + + +

+

+ which has no parameters. +

+

+ +Writing bytes. There are three calls for writing bytes to a stream. +

+

+ + + public synchronized void write(byte[] b) + + +

+

+ where: +

+
    + +
  • + +

    + +b is an array of bytes to write. +

    + +
  • + +
+

+ + + public synchronized void write(byte[] b, int off, int len) + + +

+

+ where: +

+
    + +
  • + +

    + +b is an array of bytes to write. +

    + +
  • + + +
  • + +

    + +off is a buffer offset. +

    + +
  • + + +
  • + +

    + +len is the length to write. +

    + +
  • + +
+

+ + + public synchronized void write(int b) + + +

+

+ where: +

+
    + +
  • + +

    + +b contains a byte to write. The method writes the least significant byte of the integer four bytes. +

    + +
  • + +
+
+ + + +

Reading a stream of bytes

+
+

+ Class LedgerOutputStream implements two constructors and four public methods: +

+

+ + + public LedgerInputStream(LedgerHandle lh) + throws BKException, InterruptedException + + +

+

+ where: +

+
    + +
  • + +

    + +lh is a ledger handle for a previously created and open ledger. +

    + +
  • + +
+

+ + + public LedgerInputStream(LedgerHandle lh, int size) + throws BKException, InterruptedException + + +

+

+ where: +

+
    + +
  • + +

    + +lh is a ledger handle for a previously created and open ledger. +

    + +
  • + + +
  • + +

    + +size is the size of the byte buffer to store bytes that the application + will eventually read. +

    + +
  • + +
+

+ +Closing. There is one call to close an input stream, but the call + is currently empty and the application is responsible for closing the ledger handle. +

+

+ + + public void close() + + +

+

+ which has no parameters. +

+

+ +Reading. There are three calls to read from the stream. +

+

+ + + public synchronized int read() + throws IOException + + +

+

+ which has no parameters. +

+

+ + + public synchronized int read(byte[] b) + throws IOException + + +

+

+ where: +

+
    + +
  • + +

    + +b is a byte array to write to. +

    + +
  • + +
+

+ + + public synchronized int read(byte[] b, int off, int len) + throws IOException + + +

+

+ where: +

+
    + +
  • + +

    + +b is a byte array to write to. +

    + +
  • + + +
  • + +

    + +off is an offset for byte array b. +

    + +
  • + + +
  • + +

    + +len is the length in bytes to write to b. +

    + +
  • + +
+
+ +

+ +

+
+ +
 
+
+ + + Added: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStream.pdf URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStream.pdf?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/bookkeeperStream.pdf ------------------------------------------------------------------------------ svn:mime-type = application/pdf Added: zookeeper/site/trunk/content/doc/r3.4.7/broken-links.xml URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/broken-links.xml?rev=1717709&view=auto ============================================================================== --- zookeeper/site/trunk/content/doc/r3.4.7/broken-links.xml (added) +++ zookeeper/site/trunk/content/doc/r3.4.7/broken-links.xml Thu Dec 3 04:29:41 2015 @@ -0,0 +1,2 @@ + + Added: zookeeper/site/trunk/content/doc/r3.4.7/images/2pc.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/2pc.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/2pc.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/bk-overview.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/bk-overview.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/bk-overview.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/built-with-forrest-button.png URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/built-with-forrest-button.png?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/built-with-forrest-button.png ------------------------------------------------------------------------------ svn:mime-type = image/png Added: zookeeper/site/trunk/content/doc/r3.4.7/images/favicon.ico URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/favicon.ico?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/favicon.ico ------------------------------------------------------------------------------ svn:mime-type = image/x-icon Added: zookeeper/site/trunk/content/doc/r3.4.7/images/hadoop-logo.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/hadoop-logo.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/hadoop-logo.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/instruction_arrow.png URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/instruction_arrow.png?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/instruction_arrow.png ------------------------------------------------------------------------------ svn:mime-type = image/png Added: zookeeper/site/trunk/content/doc/r3.4.7/images/state_dia.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/state_dia.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/state_dia.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/zkcomponents.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/zkcomponents.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/zkcomponents.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/zknamespace.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/zknamespace.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/zknamespace.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfRW-3.2.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfRW-3.2.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfRW-3.2.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfRW.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfRW.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfRW.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfreliability.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfreliability.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/zkperfreliability.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/zkservice.jpg URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/zkservice.jpg?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/zkservice.jpg ------------------------------------------------------------------------------ svn:mime-type = image/jpeg Added: zookeeper/site/trunk/content/doc/r3.4.7/images/zookeeper_small.gif URL: http://svn.apache.org/viewvc/zookeeper/site/trunk/content/doc/r3.4.7/images/zookeeper_small.gif?rev=1717709&view=auto ============================================================================== Binary file - no diff available. Propchange: zookeeper/site/trunk/content/doc/r3.4.7/images/zookeeper_small.gif ------------------------------------------------------------------------------ svn:mime-type = image/gif