You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@gearpump.apache.org by ma...@apache.org on 2016/09/29 12:06:25 UTC

[04/11] incubator-gearpump git commit: [GEARPUMP-213] build docs with MkDocs

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/fonts/glyphicons-halflings-regular.woff
----------------------------------------------------------------------
diff --git a/docs/fonts/glyphicons-halflings-regular.woff b/docs/fonts/glyphicons-halflings-regular.woff
deleted file mode 100644
index 9e61285..0000000
Binary files a/docs/fonts/glyphicons-halflings-regular.woff and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/fonts/glyphicons-halflings-regular.woff2
----------------------------------------------------------------------
diff --git a/docs/fonts/glyphicons-halflings-regular.woff2 b/docs/fonts/glyphicons-halflings-regular.woff2
deleted file mode 100644
index 64539b5..0000000
Binary files a/docs/fonts/glyphicons-halflings-regular.woff2 and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/gearpump-internals.md
----------------------------------------------------------------------
diff --git a/docs/gearpump-internals.md b/docs/gearpump-internals.md
deleted file mode 100644
index ba80072..0000000
--- a/docs/gearpump-internals.md
+++ /dev/null
@@ -1,234 +0,0 @@
----
-layout: global
-displayTitle: Gearpump Internals
-title: Gearpump Internals
-description: Gearpump Internals
----
-### Actor Hierarchy?
-
-![Actor Hierarchy](img/actor_hierarchy.png)
-
-Everything in the diagram is an actor; they fall into two categories, Cluster Actors and Application Actors.
-
-#### Cluster Actors
-
-  **Worker**: Maps to a physical worker machine. It is responsible for managing resources and report metrics on that machine.
-
-  **Master**: Heart of the cluster, which manages workers, resources, and applications. The main function is delegated to three child actors, App Manager, Worker Manager, and Resource Scheduler.
-
-#### Application Actors:
-
-  **AppMaster**: Responsible to schedule the tasks to workers and manage the state of the application. Different applications have different AppMaster instances and are isolated.
-
-  **Executor**: Child of AppMaster, represents a JVM process. Its job is to manage the life cycle of tasks and recover the tasks in case of failure.
-
-  **Task**: Child of Executor, does the real job. Every task actor has a global unique address. One task actor can send data to any other task actors. This gives us great flexibility of how the computation DAG is distributed.
-
-  All actors in the graph are weaved together with actor supervision, and actor watching and every error is handled properly via supervisors. In a master, a risky job is isolated and delegated to child actors, so it's more robust. In the application, an extra intermediate layer "Executor" is created so that we can do fine-grained and fast recovery in case of task failure. A master watches the lifecycle of AppMaster and worker to handle the failures, but the life cycle of Worker and AppMaster are not bound to a Master Actor by supervision, so that Master node can fail independently.  Several Master Actors form an Akka cluster, the Master state is exchanged using the Gossip protocol in a conflict-free consistent way so that there is no single point of failure. With this hierarchy design, we are able to achieve high availability.
-
-### Application Clock and Global Clock Service
-
-Global clock service will track the minimum time stamp of all pending messages in the system. Every task will update its own minimum-clock to global clock service; the minimum-clock of task is decided by the minimum of:
-
-  - Minimum time stamp of all pending messages in the inbox.
-  - Minimum time stamp of all un-acked outgoing messages. When there is message loss, the minimum clock will not advance.
-  - Minimum clock of all task states. If the state is accumulated by a lot of input messages, then the clock value is decided by the oldest message's timestamp. The state clock will advance by doing snapshots to persistent storage or by fading out the effect of old messages.
-
-![Clock](img/clock.png)
-
-The global clock service will keep track of all task minimum clocks effectively and maintain a global view of minimum clock. The global minimum clock value is monotonically increasing; it means that all source messages before this clock value have been processed. If there is message loss or task crash, the global minimum clock will stop.
-
-### How do we optimize the message passing performance?
-
-For streaming application, message passing performance is extremely important. For example, one streaming platform may need to process millions of messages per second with millisecond level latency. High throughput and low latency is not that easy to achieve. There are a number of challenges:
-
-#### First Challenge: Network is not efficient for small messages
-
-In streaming, typical message size is very small, usually less than 100 bytes per message, like the floating car GPS data. But network efficiency is very bad when transferring small messages. As you can see in below diagram, when message size is 50 bytes, it can only use 20% bandwidth. How to improve the throughput?
-
-![Throughput vs. Message Size](img/through_vs_message_size.png)
-
-#### Second Challenge: Message overhead is too big
-
-For each message sent between two actors, it contains sender and receiver actor path. When sending over the wire, the overhead of this ActorPath is not trivial. For example, the below actor path takes more than 200 bytes.
-
-```javascript
-akka.tcp://system1@192.168.1.53:51582/remote/akka.tcp/2120193a-e10b-474e-bccb-8ebc4b3a0247@192.168.1.53:48948/remote/akka.tcp/system2@192.168.1.54:43676/user/master/Worker1/app_0_executor_0/group_1_task_0#-768886794
-```
-
-#### How do we solve this?
-
-We implement a custom Netty transportation layer with Akka extension. In the below diagram, Netty Client will translate ActorPath to TaskId, and Netty Server will translate it back. Only TaskId will be passed on wire, it is only about 10 bytes, the overhead is minimized. Different Netty Client Actors are isolated; they will not block each other.
-
-![Netty Transport](img/netty_transport.png)
-
-For performance, effective batching is really the key! We group multiple messages to a single batch and send it on the wire. The batch size is not fixed; it is adjusted dynamically based on network status. If the network is available, we will flush pending messages immediately without waiting; otherwise we will put the message in a batch and trigger a timer to flush the batch later.
-
-### How do we do flow Control?
-
-Without flow control, one task can easily flood another task with too many messages, causing out of memory error. Typical flow control will use a TCP-like sliding window, so that source and target can run concurrently without blocking each other.
-
-![Flow Control](img/flow_control.png)
-Figure: Flow control, each task is "star" connected to input tasks and output tasks
-
-The difficult part for our problem is that each task can have multiple input tasks and output tasks. The input and output must be geared together so that the back pressure can be properly propagated from downstream to upstream. The flow control also needs to consider failures, and it needs to be able to recover when there is message loss.
-Another challenge is that the overhead of flow control messages can be big. If we ack every message, there will be huge amount of acked messages in the system, degrading streaming performance. The approach we adopted is to use explicit AckRequest message. The target tasks will only ack back when they receive the AckRequest message, and the source will only send AckRequest when it feels necessary. With this approach, we can largely reduce the overhead.
-
-### How do we detect message loss?
-
-For example, for web ads, we may charge for every click, we don't want to miscount.  The streaming platform needs to effectively track what messages have been lost, and recover as fast as possible.
-
-![Message Loss](img/messageLoss.png)
-Figure: Message Loss Detection
-
-We use the flow control message AckRequest and Ack to detect message loss. The target task will count how many messages has been received since last AckRequest, and ack the count back to source task. The source task will check the count and find message loss.
-This is just an illustration, the real case is more difficulty, we need to handle zombie tasks, and in-the-fly stale messages.
-
-### How Gearpump know what messages to replay?
-
-In some applications, a message cannot be lost, and must be replayed. For example, during the money transfer, the bank will SMS us the verification code. If that message is lost, the system must replay it so that money transfer can continue. We made the decision to use **source end message storage** and **time stamp based replay**.
-
-![Replay](img/replay.png)
-Figure: Replay with Source End Message Store
-
-Every message is immutable, and tagged with a timestamp. We have an assumption that the timestamp is approximately incremental (allow small ratio message disorder).
-
-We assume the message is coming from a replay-able source, like Kafka queue; otherwise the message will be stored at customizable source end "message store". When the source task sends the message downstream, the timestamp and offset of the message is also check-pointed to offset-timestamp storage periodically. During recovery, the system will first retrieve the right time stamp and offset from the offset-timestamp storage, then it will replay the message store from that time stamp and offset. A Timestamp Filter will filter out old messages in case the message in message store is not strictly time-ordered.
-
-### Master High Availability
-
-In a distributed streaming system, any part can fail. The system must stay responsive and do recovery in case of errors.
-
-![HA](img/ha.png)
-Figure: Master High Availability
-
-We use Akka clustering to implement the Master high availability. The cluster consists of several master nodes, but no worker nodes. With clustering facilities, we can easily detect and handle the failure of master node crash. The master state is replicated on all master nodes with the Typesafe akka-data-replication  library, when one master node crashes, another standby master will read the master state and take over. The master state contains the submission data of all applications. If one application dies, a master can use that state to recover that application. CRDT LwwMap  is used to represent the state; it is a hash map that can converge on distributed nodes without conflict. To have strong data consistency, the state read and write must happen on a quorum of master nodes.
-
-### How we do handle failures?
-
-With Akka's powerful actor supervision, we can implement a resilient system relatively easy. In Gearpump, different applications have a different AppMaster instance, they are totally isolated from each other. For each application, there is a supervision tree, AppMaster->Executor->Task. With this supervision hierarchy, we can free ourselves from the headache of zombie process, for example if AppMaster is down, Akka supervisor will ensure the whole tree is shutting down.
-
-There are multiple possible failure scenarios
-
-![Failures](img/failures.png)
-Figure: Possible Failure Scenarios and Error Supervision Hierarchy
-
-#### What happens when the Master crashes?
-
-In case of a master crash, other standby masters will be notified, they will resume the master state, and take over control. Worker and AppMaster will also be notified, They will trigger a process to find the new active master, until the resolution complete. If AppMaster or Worker cannot resolve a new Master in a time out, they will make suicide and kill themselves.
-
-#### What happens when a worker crashes?
-
-In case of a worker crash, the Master will get notified and stop scheduling new computation to this worker. All supervised executors on current worker will be killed, AppMaster can treat it as recovery of executor crash like [What happen when an executor crashes?](#what-happen-when-an-executor-crashes)
-
-#### What happens when the AppMaster crashes?
-
-If an AppMaster crashes, Master will schedule a new resource to create a new AppMaster Instance elsewhere, and then the AppMaster will handle the recovery inside the application. For streaming, it will recover the latest min clock and other state from disk, request resources from master to start executors, and restart the tasks with recovered min clock.
-
-#### What happen when an executor crashes?
-
-If an executor crashes, its supervisor AppMaster will get notified, and request a new resource from the active master to start a new executor, to run the tasks which were located on the crashed executor.
-
-#### What happen when tasks crash?
-
-If a task throws an exception, its supervisor executor will restart that Task.
-
-When "at least once" message delivery is enabled, it will trigger the message replaying in the case of message loss. First AppMaster will read the latest minimum clock from the global clock service(or clock storage if the clock service crashes), then AppMaster will restart all the task actors to get a fresh task state, then the source end tasks will replay messages from that minimum clock.
-
-### How does "exactly-once" message delivery work?
-
-For some applications, it is extremely important to do "exactly once" message delivery. For example, for a real-time billing system, we will not want to bill the customer twice. The goal of "exactly once" message delivery is to make sure:
-  The error doesn't accumulate, today's error will not be accumulated to tomorrow.
-  Transparent to application developer
-We use global clock to synchronize the distributed transactions. We assume every message from the data source will have a unique timestamp, the timestamp can be a part of the message body, or can be attached later with system clock when the message is injected into the streaming system. With this global synchronized clock, we can coordinate all tasks to checkpoint at same timestamp.
-
-![Checkpoint](img/checkpointing.png)
-Figure: Checkpointing and Exactly-Once Message delivery
-
-Workflow to do state checkpointing:
-
-1. The coordinator asks the streaming system to do checkpoint at timestamp Tc.
-2. For each application task, it will maintain two states, checkpoint state and current state. Checkpoint state only contains information before timestamp Tc. Current state contains all information.
-3. When global minimum clock is larger than Tc, it means all messages older than Tc has been processed; the checkpoint state will no longer change, so we will then persist the checkpoint state to storage safely.
-4. When there is message loss, we will start the recovery process.
-5. To recover, load the latest checkpoint state from store, and then use it to restore the application status.
-6. Data source replays messages from the checkpoint timestamp.
-
-The checkpoint interval is determined by global clock service dynamically. Each data source will track the max timestamp of input messages. Upon receiving min clock updates, the data source will report the time delta back to global clock service. The max time delta is the upper bound of the application state timespan. The checkpoint interval is bigger than max delta time:
-
-![Checkpoint Equation](img/checkpoint_equation.png)
-
-![Checkpointing Interval](img/checkpointing_interval.png)
-Figure: How to determine Checkpoint Interval
-
-After the checkpoint interval is notified to tasks by global clock service, each task will calculate its next checkpoint timestamp autonomously without global synchronization.
-
-![Checkpoint Interval Equation](img/checkpoint_interval_equation.png)
-
-For each task, it contains two states, checkpoint state and current state. The code to update the state is shown in listing below.
-
-```python
-TaskState(stateStore, initialTimeStamp):
-  currentState = stateStore.load(initialTimeStamp)
-  checkpointState = currentState.clone
-  checkpointTimestamp = nextCheckpointTimeStamp(initialTimeStamp)
-onMessage(msg):
-  if (msg.timestamp < checkpointTimestamp):
-    checkpointState.updateMessage(msg)
-  currentState.updateMessage(msg)  
-  maxClock = max(maxClock, msg.timeStamp)
-
-onMinClock(minClock):
-  if (minClock > checkpointTimestamp):
-    stateStore.persist(checkpointState)
-    checkpointTimeStamp = nextCheckpointTimeStamp(maxClock)
-    checkpointState = currentState.clone
-
-onNewCheckpointInterval(newStep):
-  step = newStep  
-nextCheckpointTimeStamp(timestamp):
-  checkpointTimestamp = (1 + timestamp/step) * step
-```
-
-List 1: Task Transactional State Implementation
-
-### What is dynamic graph, and how it works?
-
-The DAG can be modified dynamically. We want to be able to dynamically add, remove, and replace a sub-graph.
-
-![Dynamic DAG](img/dynamic.png)
-Figure: Dynamic Graph, Attach, Replace, and Remove
-
-## At least once message delivery and Kafka
-
-The Kafka source example project and tutorials can be found at:
-- [Kafka connector example project](https://github.com/apache/incubator-gearpump/tree/master/examples/streaming/kafka)
-- [Connect with Kafka source](dev-connectors.html)
-
-In this doc, we will talk about how the at least once message delivery works.
-
-We will use the WordCount example of [source tree](https://github.com/apache/incubator-gearpump/tree/master/examples/streaming/kafka)  to illustrate.
-
-### How the kafka WordCount DAG looks like:
-
-It contains three processors:
-![Kafka WordCount](img/kafka_wordcount.png)
-
-- KafkaStreamProducer(or KafkaSource) will read message from kafka queue.
-- Split will split lines to words
-- Sum will summarize the words to get a count for each word.
-
-### How to read data from Kafka
-
-We use KafkaSource, please check [Connect with Kafka source](dev-connectors.html) for the introduction.
-
-Please note that we have set a startTimestamp for the KafkaSource, which means KafkaSource will read from Kafka queue starting from messages whose timestamp is near startTimestamp.
-
-### What happen where there is Task crash or message loss?
-When there is message loss, the AppMaster will first pause the global clock service so that the global minimum timestamp no longer change, then it will restart the Kafka source tasks. Upon restart, Kafka Source will start to replay. It will first read the global minimum timestamp from AppMaster, and start to read message from that timestamp.
-
-### What method KafkaSource used to read messages from a start timestamp? As we know Kafka queue doesn't expose the timestamp information.
-
-Kafka queue only expose the offset information for each partition. What KafkaSource do is to maintain its own mapping from Kafka offset to  Application timestamp, so that we can map from a application timestamp to a Kafka offset, and replay Kafka messages from that Kafka offset.
-
-The mapping between Application timestamp with Kafka offset is stored in a distributed file system or as a Kafka topic.

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/get-gearpump-distribution.md
----------------------------------------------------------------------
diff --git a/docs/get-gearpump-distribution.md b/docs/get-gearpump-distribution.md
deleted file mode 100644
index 345c3d0..0000000
--- a/docs/get-gearpump-distribution.md
+++ /dev/null
@@ -1,90 +0,0 @@
----
-layout: global
-displayTitle: How to Get Your Gearpump Distribution
-title: Get Gearpump Distribution
-description: How to Get Your Gearpump Distribution
----
-
-### Prepare the binary
-You can either download pre-build release package or choose to build from source code.
-
-#### Download Release Binary
-
-If you choose to use pre-build package, then you don't need to build from source code. The release package can be downloaded from:
-
-##### [Download page](/downloads.html)
-
-#### Build from Source code
-
-If you choose to build the package from source code yourself, you can follow these steps:
-
-1). Clone the Gearpump repository
-
-```bash
-  git clone https://github.com/apache/incubator-gearpump.git
-  cd gearpump
-```
-
-2). Build package
-
-```bash
-  ## Please use scala 2.11
-  ## The target package path: output/target/gearpump-{{ site.SCALA_BINARY_VERSION }}-{{ site.GEARPUMP_VERSION }}.zip
-  sbt clean assembly packArchiveZip
-```
-
-  After the build, there will be a package file gearpump-{{ site.SCALA_BINARY_VERSION }}-{{ site.GEARPUMP_VERSION }}.zip generated under output/target/ folder.
-
-  **NOTE:**
-  Please set JAVA_HOME environment before the build.
-
-  On linux:
-
-```bash
-  export JAVA_HOME={path/to/jdk/root/path}
-```
-
-  On Windows:
-
-```bash
-  set JAVA_HOME={path/to/jdk/root/path}
-```
-
-  **NOTE:**
-The build requires network connection. If you are behind an enterprise proxy, make sure you have set the proxy in your env before running the build commands.
-For windows:
-
-```bash
-set HTTP_PROXY=http://host:port
-set HTTPS_PROXY= http://host:port
-```
-
-For Linux:
-
-```bash
-export HTTP_PROXY=http://host:port
-export HTTPS_PROXY= http://host:port
-```
-
-### Gearpump package structure
-
-You need to flatten the `.zip` file to use it. On Linux, you can
-
-```bash
-unzip gearpump-{{site.SCALA_BINARY_VERSION}}-{{site.GEARPUMP_VERSION}}.zip
-```
-
-After decompression, the directory structure looks like picture 1.
-
-![Layout](img/layout.png)
-
-Under bin/ folder, there are script files for Linux(bash script) and Windows(.bat script).
-
-script | function
---------|------------
-local | You can start the Gearpump cluster in single JVM(local mode), or in a distributed cluster(cluster mode). To start the cluster in local mode, you can use the local /local.bat helper scripts, it is very useful for developing or troubleshooting.
-master | To start Gearpump in cluster mode, you need to start one or more master nodes, which represent the global resource management center. master/master.bat is launcher script to boot the master node.
-worker | To start Gearpump in cluster mode, you also need to start several workers, with each worker represent a set of local resources. worker/worker.bat is launcher script to start the worker node.
-services | This script is used to start backend REST service and other services for frontend UI dashboard (Default user "admin, admin").
-
-Please check [Command Line Syntax](commandline.html) for more information for each script.

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/hardware-requirement.md
----------------------------------------------------------------------
diff --git a/docs/hardware-requirement.md b/docs/hardware-requirement.md
deleted file mode 100644
index 42e839c..0000000
--- a/docs/hardware-requirement.md
+++ /dev/null
@@ -1,37 +0,0 @@
----
-layout: global
-displayTitle: Gearpump Cluster Requirement
-title: Gearpump Cluster Requirement
-description: Gearpump Cluster Requirement
----
-
-### Pre-requisite
-
-Gearpump cluster can be installed on Windows OS and Linux.
-
-Before installation, you need to decide how many machines are used to run this cluster.
-
-For each machine, the requirements are listed in table below.
-
-**  Table: Environment requirement on single machine**
-
-Resource | Requirements
------------- | ---------------------------
-Memory       | 2GB free memory is required to run the cluster. For any production system, 32GB memory is recommended.
-Java	       | JRE 6 or above
-User permission | Root permission is not required
-Network	Ethernet |(TCP/IP)
-CPU	| Nothing special
-HDFS installation	| Default is not required. You only need to install it when you want to store the application jars in HDFS.
-Kafka installation |	Default is not required. You need to install Kafka when you want the at-least once message delivery feature. Currently, the only supported data source for this feature is Kafka
-
-**  Table: The default port used in Gearpump:**
-
-| usage	| Port |	Description |
------------- | ---------------|------------
-  Dashboard UI	| 8090	| Web UI.
-Dashboard web socket service |	8091 |	UI backend web socket service for long connection.
-Master port |	3000 |	Every other role like worker, appmaster, executor, user use this port to communicate with Master.
-
-You need to ensure that your firewall has not banned these ports to ensure Gearpump can work correctly.
-And you can modify the port configuration. Check [Configuration](deployment-configuration.html) section for details.  

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/actor_hierarchy.png
----------------------------------------------------------------------
diff --git a/docs/img/actor_hierarchy.png b/docs/img/actor_hierarchy.png
deleted file mode 100644
index d971745..0000000
Binary files a/docs/img/actor_hierarchy.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/checkpoint_equation.png
----------------------------------------------------------------------
diff --git a/docs/img/checkpoint_equation.png b/docs/img/checkpoint_equation.png
deleted file mode 100644
index 14da93b..0000000
Binary files a/docs/img/checkpoint_equation.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/checkpoint_interval_equation.png
----------------------------------------------------------------------
diff --git a/docs/img/checkpoint_interval_equation.png b/docs/img/checkpoint_interval_equation.png
deleted file mode 100644
index 0c0414c..0000000
Binary files a/docs/img/checkpoint_interval_equation.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/checkpointing.png
----------------------------------------------------------------------
diff --git a/docs/img/checkpointing.png b/docs/img/checkpointing.png
deleted file mode 100644
index f11eb53..0000000
Binary files a/docs/img/checkpointing.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/checkpointing_interval.png
----------------------------------------------------------------------
diff --git a/docs/img/checkpointing_interval.png b/docs/img/checkpointing_interval.png
deleted file mode 100644
index dc46317..0000000
Binary files a/docs/img/checkpointing_interval.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/clock.png
----------------------------------------------------------------------
diff --git a/docs/img/clock.png b/docs/img/clock.png
deleted file mode 100644
index 906d51d..0000000
Binary files a/docs/img/clock.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/dag.png
----------------------------------------------------------------------
diff --git a/docs/img/dag.png b/docs/img/dag.png
deleted file mode 100644
index c0ca79f..0000000
Binary files a/docs/img/dag.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/dashboard.gif
----------------------------------------------------------------------
diff --git a/docs/img/dashboard.gif b/docs/img/dashboard.gif
deleted file mode 100644
index 0170c5f..0000000
Binary files a/docs/img/dashboard.gif and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/dashboard.png
----------------------------------------------------------------------
diff --git a/docs/img/dashboard.png b/docs/img/dashboard.png
deleted file mode 100644
index 0b5eedd..0000000
Binary files a/docs/img/dashboard.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/dashboard_3.png
----------------------------------------------------------------------
diff --git a/docs/img/dashboard_3.png b/docs/img/dashboard_3.png
deleted file mode 100644
index 47259fc..0000000
Binary files a/docs/img/dashboard_3.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/download.jpg
----------------------------------------------------------------------
diff --git a/docs/img/download.jpg b/docs/img/download.jpg
deleted file mode 100644
index 7129c52..0000000
Binary files a/docs/img/download.jpg and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/dynamic.png
----------------------------------------------------------------------
diff --git a/docs/img/dynamic.png b/docs/img/dynamic.png
deleted file mode 100644
index 09b8a35..0000000
Binary files a/docs/img/dynamic.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/exact.png
----------------------------------------------------------------------
diff --git a/docs/img/exact.png b/docs/img/exact.png
deleted file mode 100644
index f11eb53..0000000
Binary files a/docs/img/exact.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/failures.png
----------------------------------------------------------------------
diff --git a/docs/img/failures.png b/docs/img/failures.png
deleted file mode 100644
index fa98cdc..0000000
Binary files a/docs/img/failures.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/flow_control.png
----------------------------------------------------------------------
diff --git a/docs/img/flow_control.png b/docs/img/flow_control.png
deleted file mode 100644
index 7ea9bd2..0000000
Binary files a/docs/img/flow_control.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/flowcontrol.png
----------------------------------------------------------------------
diff --git a/docs/img/flowcontrol.png b/docs/img/flowcontrol.png
deleted file mode 100644
index 7ea9bd2..0000000
Binary files a/docs/img/flowcontrol.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/ha.png
----------------------------------------------------------------------
diff --git a/docs/img/ha.png b/docs/img/ha.png
deleted file mode 100644
index 5474d84..0000000
Binary files a/docs/img/ha.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/kafka_wordcount.png
----------------------------------------------------------------------
diff --git a/docs/img/kafka_wordcount.png b/docs/img/kafka_wordcount.png
deleted file mode 100644
index a43fa55..0000000
Binary files a/docs/img/kafka_wordcount.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/layout.png
----------------------------------------------------------------------
diff --git a/docs/img/layout.png b/docs/img/layout.png
deleted file mode 100644
index edffdf8..0000000
Binary files a/docs/img/layout.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/logo.png
----------------------------------------------------------------------
diff --git a/docs/img/logo.png b/docs/img/logo.png
deleted file mode 100644
index 7575892..0000000
Binary files a/docs/img/logo.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/logo.svg
----------------------------------------------------------------------
diff --git a/docs/img/logo.svg b/docs/img/logo.svg
deleted file mode 100644
index 5897ca4..0000000
--- a/docs/img/logo.svg
+++ /dev/null
@@ -1,71 +0,0 @@
-<?xml version="1.0" encoding="UTF-8" standalone="no"?>
-<!-- Created with Inkscape (http://www.inkscape.org/) -->
-
-<svg
-   xmlns:dc="http://purl.org/dc/elements/1.1/"
-   xmlns:cc="http://creativecommons.org/ns#"
-   xmlns:rdf="http://www.w3.org/1999/02/22-rdf-syntax-ns#"
-   xmlns:svg="http://www.w3.org/2000/svg"
-   xmlns="http://www.w3.org/2000/svg"
-   xmlns:sodipodi="http://sodipodi.sourceforge.net/DTD/sodipodi-0.dtd"
-   xmlns:inkscape="http://www.inkscape.org/namespaces/inkscape"
-   width="89mm"
-   height="89mm"
-   viewBox="0 0 89 89"
-   id="svg4684"
-   version="1.1"
-   inkscape:version="0.91 r13725"
-   sodipodi:docname="gearpump_logo_simple.svg"
-   inkscape:export-filename="/Users/kamkasravi/Dropbox/gearpump/gearpump_logo_simple.png"
-   inkscape:export-xdpi="157.11"
-   inkscape:export-ydpi="157.11">
-  <defs
-     id="defs4686" />
-  <sodipodi:namedview
-     inkscape:document-units="mm"
-     pagecolor="#ffffff"
-     bordercolor="#666666"
-     borderopacity="0"
-     inkscape:pageopacity="0"
-     inkscape:pageshadow="2"
-     inkscape:zoom="1.2162434"
-     inkscape:cx="395.86179"
-     inkscape:cy="372.04726"
-     inkscape:current-layer="layer1"
-     id="namedview4688"
-     showgrid="false"
-     inkscape:window-width="1920"
-     inkscape:window-height="1107"
-     inkscape:window-x="0"
-     inkscape:window-y="1"
-     inkscape:window-maximized="1"
-     fit-margin-top="0"
-     fit-margin-left="0"
-     fit-margin-right="0"
-     fit-margin-bottom="0"
-     inkscape:showpageshadow="false"
-     showborder="true" />
-  <metadata
-     id="metadata4690">
-    <rdf:RDF>
-      <cc:Work
-         rdf:about="">
-        <dc:format>image/svg+xml</dc:format>
-        <dc:type
-           rdf:resource="http://purl.org/dc/dcmitype/StillImage" />
-        <dc:title />
-      </cc:Work>
-    </rdf:RDF>
-  </metadata>
-  <g
-     inkscape:label="Layer 1"
-     inkscape:groupmode="layer"
-     id="layer1"
-     transform="translate(5.4566873e-7,-121)">
-    <path
-       style="fill:#556ae6;fill-opacity:1;stroke:none;stroke-width:7.08661413;stroke-miterlimit:4;stroke-dasharray:none"
-       d="m 59.451523,122.95775 a 44.775326,44.775326 0 0 0 -21.03861,-2.01136 l -0.6557,4.60251 a 23.524902,23.524902 0 0 1 4.20498,3.77358 l 6.73498,-1.78044 a 29.907628,29.907628 0 0 1 4.33792,8.99185 l -5.54364,4.16221 a 23.524902,23.524902 0 0 1 0.32108,5.63603 l 6.03185,3.50988 a 29.907628,29.907628 0 0 1 -1.23578,4.84982 29.907628,29.907628 0 0 1 -2.06484,4.5672 l -6.85567,-0.97589 a 23.524902,23.524902 0 0 1 -3.77339,4.20446 l 1.78078,6.73569 a 29.907628,29.907628 0 0 1 -8.99151,4.33863 l -4.16256,-5.54435 a 23.524902,23.524902 0 0 1 -5.63602,0.32108 l -3.50989,6.03185 a 29.907628,29.907628 0 0 1 -4.849808,-1.23578 29.907628,29.907628 0 0 1 -4.5675477,-2.06554 l 0.9757077,-6.85515 a 23.524902,23.524902 0 0 1 -4.2041157,-3.7727 l -6.64489998,1.75628 a 44.775326,44.775326 0 0 0 29.97375368,45.3569 44.775326,44.775326 0 0 0 27.28111,0.61075 l -0.0231,-0.19065 a 23.524902,23.524902 0 0 1 -5.04173,-2.54001 l -6.05573,3.46919 a 29.907628,29.907628 0 0 1 -3.58191,-3.49598 29.907628
 ,29.907628 0 0 1 -2.9233,-4.07194 l 4.27314,-5.44883 a 23.524902,23.524902 0 0 1 -1.75454,-5.36936 l -6.72324,-1.82616 a 29.907628,29.907628 0 0 1 0.73798,-9.9567 l 6.88304,-0.8322 a 23.524902,23.524902 0 0 1 2.54001,-5.04173 l -3.46885,-6.05502 a 29.907628,29.907628 0 0 1 3.49494,-3.58228 29.907628,29.907628 0 0 1 4.07194,-2.92331 l 5.44901,4.27262 a 23.524902,23.524902 0 0 1 5.36988,-1.75436 l 1.82616,-6.72324 a 29.907628,29.907628 0 0 1 9.95599,0.73833 l 0.8329,6.8827 a 23.524902,23.524902 0 0 1 5.04121,2.53983 l 6.055556,-3.46867 a 29.907628,29.907628 0 0 1 3.58227,3.49494 29.907628,29.907628 0 0 1 1.52975,1.98522 44.775326,44.775326 0 0 0 -29.979576,-45.3099 z m -29.93126,7.04882 a 15.745845,15.745845 0 0 0 -20.0396247,9.71017 15.745845,15.745845 0 0 0 9.7101647,20.03962 15.745845,15.745845 0 0 0 20.03963,-9.71016 15.745845,15.745845 0 0 0 -9.71017,-20.03963 z m 59.360536,42.82725 -2.32895,2.97009 a 23.524902,23.524902 0 0 1 0.99553,2.56761 44.775326,44.775326 0 0 0 1.33342,-5.
 5377 z m -19.143966,-2.09047 a 15.745845,15.745845 0 0 0 -14.75675,3.19169 15.745845,15.745845 0 0 0 -1.61104,22.20982 15.745845,15.745845 0 0 0 21.90246,1.85092 44.775326,44.775326 0 0 0 2.25015,-2.25582 15.745845,15.745845 0 0 0 -0.3321,-20.19458 15.745845,15.745845 0 0 0 -7.45272,-4.80203 z"
-       id="path5303"
-       inkscape:connector-curvature="0" />
-  </g>
-</svg>

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/logo2.png
----------------------------------------------------------------------
diff --git a/docs/img/logo2.png b/docs/img/logo2.png
deleted file mode 100644
index 959d39e..0000000
Binary files a/docs/img/logo2.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/messageLoss.png
----------------------------------------------------------------------
diff --git a/docs/img/messageLoss.png b/docs/img/messageLoss.png
deleted file mode 100644
index 80b330a..0000000
Binary files a/docs/img/messageLoss.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/netty_transport.png
----------------------------------------------------------------------
diff --git a/docs/img/netty_transport.png b/docs/img/netty_transport.png
deleted file mode 100644
index 17d57c3..0000000
Binary files a/docs/img/netty_transport.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/replay.png
----------------------------------------------------------------------
diff --git a/docs/img/replay.png b/docs/img/replay.png
deleted file mode 100644
index 8bbbc43..0000000
Binary files a/docs/img/replay.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/shuffle.png
----------------------------------------------------------------------
diff --git a/docs/img/shuffle.png b/docs/img/shuffle.png
deleted file mode 100644
index 40c4a2d..0000000
Binary files a/docs/img/shuffle.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/storm_gearpump_cluster.png
----------------------------------------------------------------------
diff --git a/docs/img/storm_gearpump_cluster.png b/docs/img/storm_gearpump_cluster.png
deleted file mode 100644
index d318623..0000000
Binary files a/docs/img/storm_gearpump_cluster.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/storm_gearpump_dag.png
----------------------------------------------------------------------
diff --git a/docs/img/storm_gearpump_dag.png b/docs/img/storm_gearpump_dag.png
deleted file mode 100644
index 24920f7..0000000
Binary files a/docs/img/storm_gearpump_dag.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/submit.png
----------------------------------------------------------------------
diff --git a/docs/img/submit.png b/docs/img/submit.png
deleted file mode 100644
index 609c0d7..0000000
Binary files a/docs/img/submit.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/submit2.png
----------------------------------------------------------------------
diff --git a/docs/img/submit2.png b/docs/img/submit2.png
deleted file mode 100644
index d3939ee..0000000
Binary files a/docs/img/submit2.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/img/through_vs_message_size.png
----------------------------------------------------------------------
diff --git a/docs/img/through_vs_message_size.png b/docs/img/through_vs_message_size.png
deleted file mode 100644
index a98c528..0000000
Binary files a/docs/img/through_vs_message_size.png and /dev/null differ

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/index.md
----------------------------------------------------------------------
diff --git a/docs/index.md b/docs/index.md
deleted file mode 100644
index d24f3e4..0000000
--- a/docs/index.md
+++ /dev/null
@@ -1,37 +0,0 @@
----
-layout: global
-displayTitle: Gearpump Overview
-title: Overview
-description: Gearpump GEARPUMP_VERSION documentation homepage
----
-
-Gearpump is a real-time big data streaming engine.
-It is inspired by recent advances in the [Akka](http://akka.io/) framework and a desire to improve on existing streaming frameworks.
-Gearpump is event/message based and featured as low latency handling, high performance, exactly once semantics,
-dynamic topology update, [Apache Storm](https://storm.apache.org/) compatibility, etc.
-
-The	name	Gearpump	is	a	reference to	the	engineering term "gear	pump,"	which	is	a	super simple
-pump	that	consists of	only	two	gears,	but	is	very	powerful at	streaming water.
-
-![Logo](img/logo2.png)
-
-### Gearpump Technical Highlights
-Gearpump's feature set includes:
-
-* Extremely high performance
-* Low latency
-* Configurable message delivery guarantee (at least once, exactly once).
-* Highly extensible
-* Dynamic DAG
-* Storm compatibility
-* Samoa compatibility
-* Both high level and low level API
-
-### Gearpump Performance
-Per initial benchmarks we are able to process 18 million messages/second (100 bytes per message) with a 8ms latency on a 4-node cluster.
-
-![Dashboard](img/dashboard.png)
-
-### Gearpump and Akka
-Gearpump is a 100% Akka based platform. We model big data streaming within the Akka actor hierarchy.
-![Actor Hierarchy](img/actor_hierarchy.png)

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/js/api-docs.js
----------------------------------------------------------------------
diff --git a/docs/js/api-docs.js b/docs/js/api-docs.js
deleted file mode 100644
index 864502c..0000000
--- a/docs/js/api-docs.js
+++ /dev/null
@@ -1,43 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one or more
- * contributor license agreements.  See the NOTICE file distributed with
- * this work for additional information regarding copyright ownership.
- * The ASF licenses this file to You under the Apache License, Version 2.0
- * (the "License"); you may not use this file except in compliance with
- * the License.  You may obtain a copy of the License at
- *
- *    http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-
-/* Dynamically injected post-processing code for the API docs */
-
-$(document).ready(function () {
-  var annotations = $("dt:contains('Annotations')").next("dd").children("span.name");
-  addBadges(annotations, "AlphaComponent", ":: AlphaComponent ::", '<span class="alphaComponent badge">Alpha Component</span>');
-  addBadges(annotations, "DeveloperApi", ":: DeveloperApi ::", '<span class="developer badge">Developer API</span>');
-  addBadges(annotations, "Experimental", ":: Experimental ::", '<span class="experimental badge">Experimental</span>');
-});
-
-function addBadges(allAnnotations, name, tag, html) {
-  var annotations = allAnnotations.filter(":contains('" + name + "')")
-  var tags = $(".cmt:contains(" + tag + ")")
-
-  // Remove identifier tags from comments
-  tags.each(function (index) {
-    var oldHTML = $(this).html();
-    var newHTML = oldHTML.replace(tag, "");
-    $(this).html(newHTML);
-  });
-
-  // Add badges to all containers
-  tags.prevAll("h4.signature")
-    .add(annotations.closest("div.fullcommenttop"))
-    .add(annotations.closest("div.fullcomment").prevAll("h4.signature"))
-    .prepend(html);
-}

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/js/api-javadocs.js
----------------------------------------------------------------------
diff --git a/docs/js/api-javadocs.js b/docs/js/api-javadocs.js
deleted file mode 100644
index c4eb8a6..0000000
--- a/docs/js/api-javadocs.js
+++ /dev/null
@@ -1,60 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one or more
- * contributor license agreements.  See the NOTICE file distributed with
- * this work for additional information regarding copyright ownership.
- * The ASF licenses this file to You under the Apache License, Version 2.0
- * (the "License"); you may not use this file except in compliance with
- * the License.  You may obtain a copy of the License at
- *
- *    http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-
-/* Dynamically injected post-processing code for the API docs */
-
-$(document).ready(function () {
-  addBadges(":: AlphaComponent ::", '<span class="alphaComponent badge">Alpha Component</span>');
-  addBadges(":: DeveloperApi ::", '<span class="developer badge">Developer API</span>');
-  addBadges(":: Experimental ::", '<span class="experimental badge">Experimental</span>');
-});
-
-function addBadges(tag, html) {
-  var tags = $(".block:contains(" + tag + ")")
-
-  // Remove identifier tags
-  tags.each(function (index) {
-    var oldHTML = $(this).html();
-    var newHTML = oldHTML.replace(tag, "");
-    $(this).html(newHTML);
-  });
-
-  // Add html badge tags
-  tags.each(function (index) {
-    if ($(this).parent().is('td.colLast')) {
-      $(this).parent().prepend(html);
-    } else if ($(this).parent('li.blockList')
-        .parent('ul.blockList')
-        .parent('div.description')
-        .parent().is('div.contentContainer')) {
-      var contentContainer = $(this).parent('li.blockList')
-        .parent('ul.blockList')
-        .parent('div.description')
-        .parent('div.contentContainer')
-      var header = contentContainer.prev('div.header');
-      if (header.length > 0) {
-        header.prepend(html);
-      } else {
-        contentContainer.prepend(html);
-      }
-    } else if ($(this).parent().is('li.blockList')) {
-      $(this).parent().prepend(html);
-    } else {
-      $(this).prepend(html);
-    }
-  });
-}

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/js/main.js
----------------------------------------------------------------------
diff --git a/docs/js/main.js b/docs/js/main.js
deleted file mode 100644
index f70bbc1..0000000
--- a/docs/js/main.js
+++ /dev/null
@@ -1,108 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one or more
- * contributor license agreements.  See the NOTICE file distributed with
- * this work for additional information regarding copyright ownership.
- * The ASF licenses this file to You under the Apache License, Version 2.0
- * (the "License"); you may not use this file except in compliance with
- * the License.  You may obtain a copy of the License at
- *
- *    http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-
-/* Custom JavaScript code in the MarkDown docs */
-
-// Enable language-specific code tabs
-function codeTabs() {
-  var counter = 0;
-  var langImages = {
-    "scala": "img/scala-sm.png",
-    "python": "img/python-sm.png",
-    "java": "img/java-sm.png"
-  };
-  $("div.codetabs").each(function () {
-    $(this).addClass("tab-content");
-
-    // Insert the tab bar
-    var tabBar = $('<ul class="nav nav-tabs" data-tabs="tabs"></ul>');
-    $(this).before(tabBar);
-
-    // Add each code sample to the tab bar:
-    var codeSamples = $(this).children("div");
-    codeSamples.each(function () {
-      $(this).addClass("tab-pane");
-      var lang = $(this).data("lang");
-      var image = $(this).data("image");
-      var label = $(this).data("label");
-      var notabs = $(this).data("notabs");
-      if (label == null) {
-        var capitalizedLang = lang.substr(0, 1).toUpperCase() + lang.substr(1);
-        label = capitalizedLang;
-      }
-      lang = lang.replace(/ /g, '');
-      var id = "tab_" + label.replace(/ /g, '_') + "_" + counter;
-      $(this).attr("id", id);
-      if (image != null && langImages[lang]) {
-        var buttonLabel = "<img src='" + langImages[lang] + "' alt='" + label + "' />";
-      } else if (notabs == null) {
-        var buttonLabel = "<b>" + label + "</b>";
-      } else {
-        var buttonLabel = ""
-      }
-      tabBar.append(
-        '<li><a href="#' + id + '">' + buttonLabel + '</a></li>'
-      );
-    });
-
-    codeSamples.first().addClass("active");
-    tabBar.children("li").first().addClass("active");
-    counter++;
-  });
-  $("ul.nav-tabs a").click(function (e) {
-    // Toggling a tab should switch all tabs corresponding to the same language
-    // while retaining the scroll position
-    e.preventDefault();
-    var scrollOffset = $(this).offset().top - $(document).scrollTop();
-    $(this).tab('show');
-    $(document).scrollTop($(this).offset().top - scrollOffset);
-  });
-  $("table").each(function () {
-    $(this).addClass("table table-bordered");
-  });
-}
-
-// A script to fix internal hash links because we have an overlapping top bar.
-// Based on https://github.com/twitter/bootstrap/issues/193#issuecomment-2281510
-function maybeScrollToHash() {
-  if (window.location.hash && $(window.location.hash).length) {
-    var newTop = $(window.location.hash).offset().top - 57;
-    $(window).scrollTop(newTop);
-  }
-}
-
-$(function () {
-  codeTabs();
-  // Display anchor links when hovering over headers. For documentation of the
-  // configuration options, see the AnchorJS documentation.
-  anchors.options = {
-    placement: 'left'
-  };
-  anchors.add();
-
-  $(window).bind('hashchange', function () {
-    maybeScrollToHash();
-  });
-
-  // Scroll now too in case we had opened the page on a hash, but wait a bit because some browsers
-  // will try to do *their* initial scroll after running the onReady handler.
-  $(window).load(function () {
-    setTimeout(function () {
-      maybeScrollToHash();
-    }, 25);
-  });
-});

http://git-wip-us.apache.org/repos/asf/incubator-gearpump/blob/761e04c6/docs/js/vendor/anchor-1.1.1.min.js
----------------------------------------------------------------------
diff --git a/docs/js/vendor/anchor-1.1.1.min.js b/docs/js/vendor/anchor-1.1.1.min.js
deleted file mode 100644
index 68c3cb7..0000000
--- a/docs/js/vendor/anchor-1.1.1.min.js
+++ /dev/null
@@ -1,6 +0,0 @@
-/*!
- * AnchorJS - v1.1.1 - 2015-05-23
- * https://github.com/bryanbraun/anchorjs
- * Copyright (c) 2015 Bryan Braun; Licensed MIT
- */
-function AnchorJS(A){"use strict";this.options=A||{},this._applyRemainingDefaultOptions=function(A){this.options.icon=this.options.hasOwnProperty("icon")?A.icon:"&#xe9cb",this.options.visible=this.options.hasOwnProperty("visible")?A.visible:"hover",this.options.placement=this.options.hasOwnProperty("placement")?A.placement:"right",this.options.class=this.options.hasOwnProperty("class")?A.class:""},this._applyRemainingDefaultOptions(A),this.add=function(A){var e,t,o,n,i,s,a,l,c,r,h,g,B,Q;if(this._applyRemainingDefaultOptions(this.options),A){if("string"!=typeof A)throw new Error("The selector provided to AnchorJS was invalid.")}else A="h1, h2, h3, h4, h5, h6";if(e=document.querySelectorAll(A),0===e.length)return!1;for(this._addBaselineStyles(),t=document.querySelectorAll("[id]"),o=[].map.call(t,function(A){return A.id}),i=0;i<e.length;i++){if(e[i].hasAttribute("id"))n=e[i].getAttribute("id");else{s=e[i].textContent,a=s.replace(/[^\w\s-]/gi,"").replace(/\s+/g,"-").replace(/-{2,}/g,"-"
 ).substring(0,64).replace(/^-+|-+$/gm,"").toLowerCase(),r=a,c=0;do void 0!==l&&(r=a+"-"+c),l=o.indexOf(r),c+=1;while(-1!==l);l=void 0,o.push(r),e[i].setAttribute("id",r),n=r}h=n.replace(/-/g," "),g='<a class="anchorjs-link '+this.options.class+'" href="#'+n+'" aria-label="Anchor link for: '+h+'" data-anchorjs-icon="'+this.options.icon+'"></a>',B=document.createElement("div"),B.innerHTML=g,Q=B.childNodes,"always"===this.options.visible&&(Q[0].style.opacity="1"),"&#xe9cb"===this.options.icon&&(Q[0].style.fontFamily="anchorjs-icons",Q[0].style.fontStyle="normal",Q[0].style.fontVariant="normal",Q[0].style.fontWeight="normal"),"left"===this.options.placement?(Q[0].style.position="absolute",Q[0].style.marginLeft="-1em",Q[0].style.paddingRight="0.5em",e[i].insertBefore(Q[0],e[i].firstChild)):(Q[0].style.paddingLeft="0.375em",e[i].appendChild(Q[0]))}return this},this.remove=function(A){for(var e,t=document.querySelectorAll(A),o=0;o<t.length;o++)e=t[o].querySelector(".anchorjs-link"),e&&t[o]
 .removeChild(e);return this},this._addBaselineStyles=function(){if(null===document.head.querySelector("style.anchorjs")){var A,e=document.createElement("style"),t=" .anchorjs-link {   opacity: 0;   text-decoration: none;   -webkit-font-smoothing: antialiased;   -moz-osx-font-smoothing: grayscale; }",o=" *:hover > .anchorjs-link, .anchorjs-link:focus  {   opacity: 1; }",n=' @font-face {   font-family: "anchorjs-icons";   font-style: normal;   font-weight: normal;   src: url(data:application/x-font-ttf;charset=utf-8;base64,AAEAAAALAIAAAwAwT1MvMg8SBTUAAAC8AAAAYGNtYXAWi9QdAAABHAAAAFRnYXNwAAAAEAAAAXAAAAAIZ2x5Zgq29TcAAAF4AAABNGhlYWQEZM3pAAACrAAAADZoaGVhBhUDxgAAAuQAAAAkaG10eASAADEAAAMIAAAAFGxvY2EAKACuAAADHAAAAAxtYXhwAAgAVwAAAygAAAAgbmFtZQ5yJ3cAAANIAAAB2nBvc3QAAwAAAAAFJAAAACAAAwJAAZAABQAAApkCzAAAAI8CmQLMAAAB6wAzAQkAAAAAAAAAAAAAAAAAAAABEAAAAAAAAAAAAAAAAAAAAABAAADpywPA/8AAQAPAAEAAAAABAAAAAAAAAAAAAAAgAAAAAAADAAAAAwAAABwAAQADAAAAHAADAAEAAAAcAAQAOAAAAAoACAACAAIAAQAg6cv//f//AAAAAAAg6cv//f//AAH/4x
 Y5AAMAAQAAAAAAAAAAAAAAAQAB//8ADwABAAAAAAAAAAAAAgAANzkBAAAAAAEAAAAAAAAAAAACAAA3OQEAAAAAAQAAAAAAAAAAAAIAADc5AQAAAAACADEARAJTAsAAKwBUAAABIiYnJjQ/AT4BMzIWFxYUDwEGIicmND8BNjQnLgEjIgYPAQYUFxYUBw4BIwciJicmND8BNjIXFhQPAQYUFx4BMzI2PwE2NCcmNDc2MhcWFA8BDgEjARQGDAUtLXoWOR8fORYtLTgKGwoKCjgaGg0gEhIgDXoaGgkJBQwHdR85Fi0tOAobCgoKOBoaDSASEiANehoaCQkKGwotLXoWOR8BMwUFLYEuehYXFxYugC44CQkKGwo4GkoaDQ0NDXoaShoKGwoFBe8XFi6ALjgJCQobCjgaShoNDQ0NehpKGgobCgoKLYEuehYXAAEAAAABAACiToc1Xw889QALBAAAAAAA0XnFFgAAAADRecUWAAAAAAJTAsAAAAAIAAIAAAAAAAAAAQAAA8D/wAAABAAAAAAAAlMAAQAAAAAAAAAAAAAAAAAAAAUAAAAAAAAAAAAAAAACAAAAAoAAMQAAAAAACgAUAB4AmgABAAAABQBVAAIAAAAAAAIAAAAAAAAAAAAAAAAAAAAAAAAADgCuAAEAAAAAAAEADgAAAAEAAAAAAAIABwCfAAEAAAAAAAMADgBLAAEAAAAAAAQADgC0AAEAAAAAAAUACwAqAAEAAAAAAAYADgB1AAEAAAAAAAoAGgDeAAMAAQQJAAEAHAAOAAMAAQQJAAIADgCmAAMAAQQJAAMAHABZAAMAAQQJAAQAHADCAAMAAQQJAAUAFgA1AAMAAQQJAAYAHACDAAMAAQQJAAoANAD4YW5jaG9yanMtaWNvbnMAYQBuAGMAaABvAHIAagBzAC0AaQBjAG8AbgBzVmVyc2lvbiAxLjAAVgBlAHIAcwBpAG8AbgAgADEALgAwYW5jaG9yanMtaWN
 vbnMAYQBuAGMAaABvAHIAagBzAC0AaQBjAG8AbgBzYW5jaG9yanMtaWNvbnMAYQBuAGMAaABvAHIAagBzAC0AaQBjAG8AbgBzUmVndWxhcgBSAGUAZwB1AGwAYQByYW5jaG9yanMtaWNvbnMAYQBuAGMAaABvAHIAagBzAC0AaQBjAG8AbgBzRm9udCBnZW5lcmF0ZWQgYnkgSWNvTW9vbi4ARgBvAG4AdAAgAGcAZQBuAGUAcgBhAHQAZQBkACAAYgB5ACAASQBjAG8ATQBvAG8AbgAuAAAAAwAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAAA==) format("truetype"); }',i=" [data-anchorjs-icon]::after {   content: attr(data-anchorjs-icon); }";e.className="anchorjs",e.appendChild(document.createTextNode("")),A=document.head.querySelector('[rel="stylesheet"], style'),void 0===A?document.head.appendChild(e):document.head.insertBefore(e,A),e.sheet.insertRule(t,e.sheet.cssRules.length),e.sheet.insertRule(o,e.sheet.cssRules.length),e.sheet.insertRule(i,e.sheet.cssRules.length),e.sheet.insertRule(n,e.sheet.cssRules.length)}}}var anchors=new AnchorJS;