diff --git a/secondary/docs/design/markdown/index_manager.md b/secondary/docs/design/markdown/index_manager.md index f157f599c..39bc838af 100644 --- a/secondary/docs/design/markdown/index_manager.md +++ b/secondary/docs/design/markdown/index_manager.md @@ -1 +1,86 @@ -##Index Manager +## IndexManager + +IndexManager is the component that co-ordinate other components - like +projector, query, indexer nodes during bootstrap, rollback, reconnection, +local-indexer-restart etc. + +Since rest of the system depends on IndexManager for both normal operation and +for failure recovery, restart etc. it can end up becoming single-point-of +failure within the system. To avoid this, multiple instance of IndexManager +will be running on different nodes, where one of them will be elected as +master, called Index-Coordinator, and others will act as replica, called +Index-Coordinator-Replica, to the current master. + +### StateContext + +State context acts as the reference point for rest of the system. Typically it +contains fields to manage index DDLs, topology, event-publisher etc. +Several API will be exposed by Index-Coordinator to update StateContext or +portion of StateContext. + +### scope of IndexManager + +1. handle scan and query request from client SDKs and N1QL clients. +2. co-operate with Index-Coordinator to generate stable scan. +3. co-operate with ns-server for master election and master notification. + +### scope of Index-Coordinator + +1. save and restore StateContext from persistent storage. +2. hand-shake with local-indexer-nodes confirming the topology for each index. +3. process index DDLs. +4. for new index, generate a topology based on, + * administrator supplied configuration. + * list of local-indexer-nodes and load handled by each of them. +5. co-ordinate index re-balance. +6. generate and publish persistence timestamps to local-indexer-nodes. + * maintain a history of persistence timestamps for each bucket. +7. replicate changes in StateContext to other Index-Coordinator-Replica. +8. add, delete topics in pub-sub. subscribe, un-subscribe nodes from topics, + optionally based on predicates. +9. provide network API to other components to access index-metadata, + index topology and publish-subscribe framework. +10. generate restart timestamp for upr-reconnection. +11. negotiation with UPR producer for failover-log and restart sequence number. +12. create rollback context for kv-rollback and update rollback context based + on how rollback evolves within the system. + +### scope of ns-server + +1. master-election is done by ns-server +2. master-election is done by ns-server during system start and whenever current + master fail to respond for `hearbeat` request. +3. ns-server will be the central component that maintain the current master and + list of active replica and list of indexer-nodes. +4. actively poll - master node, replica nodes and other local-indexer-nodes for + its liveliness using `heartbeat` request. +5. provide API for IndexManagers and local-indexers to join or leave the cluster, + to fetch current Index-Coordinator, Index-Coordinator-Replicas and list of + indexer-nodes, and transaction API for updating StateContext. + +### a note on topology + +A collection of local-indexer nodes take part in building and servicing +secondary index. For any given index a subset of local-indexer nodes will be +responsible for building the index, some of them acting as master and few others +acting as active replicas. + +Topology of any given index consist of the following elements, + +* list of indexer-nodes, aka local-indexer-nodes, hosting the index. +* index slice, where each slice will hold a subset of index. +* index partition, where each partition is hosted by a master-local-indexer and + zero or more replica-local-indexer. Each partition contains a collection of + one or more slices. + +### master election + +We expect ns-server to elect a new master, + +* during system bootstrap. +* during a master crash (when master fails to respond for heartbeat request). +* when a master voluntarily leaves the cluster. + +after a new master is elected, ns-server should post a bootstrap request to +each IndexManager. There after IndexManager can fetch the current master from +ns-server and become an Index-Coordinator or Index-Coordinator-Replica. diff --git a/secondary/docs/design/markdown/index_manager_design.md b/secondary/docs/design/markdown/index_manager_design.md new file mode 100644 index 000000000..c8fcd2d72 --- /dev/null +++ b/secondary/docs/design/markdown/index_manager_design.md @@ -0,0 +1,490 @@ +## IndexManager design + +Each instance of IndexManager will be modeled as a state machine backed by a +data structure, called StateContext, that contains meta-data about the secondary +index, index topology, pub-sub data and meta-data for normal operation of +IndexManager cluster. + +``` + + *--------------* + | IndexManager | + *--------------* + | + | (program start, new-master-elected) + V + elected *-----------* + master *------------| bootstrap |-------------* (not a master) + | *-----------* | + V V + *---------------* *-----------* + | IMCoordinator | | IMReplica | + *---------------* *-----------* +``` + +**TODO: Replace it with a pictorial diagram** + +**main data structure that backs Index-Coordinator** + +```go + type Timestamp []uint64 // timestamp vector for vbuckets + + type Node struct { + name string + connectionAddr string + role []string // list of roles performed by the node. + } + + type IndexInfo struct { + Name string // Name of the index + Uuid uint64 // unique id for every index + Using IndexType // indexing algorithm + OnExprList []string // expression list + Bucket string // bucket name + IsPrimary bool + Exprtype ExprType + Active bool // whether index topology is created and index is ready + } + + type IndexTopology struct { + indexinfo IndexInfo + numPartitions int + servers []Nodes // servers hosting this index + // server can be master or replica, where, len(partitionMap) == numPartitions + // first integer-value in each map will index to master-server in + // `servers` field, and remaining elements will point to replica-servers + partitionMap [][MAX_REPLICAS+1]int + sliceMap [MAX_SLICES][MAX_REPLICAS+1]int + } + + type StateContext struct { + // value gets incremented after every updates. + cas uint64 + + // maximum number of persistence timestamp to maintain. + ptsMaxHistory int + + // per bucket timestamp continuously updated by sync message, and + // promoted to persistence timestamp. + currentTimestamp map[string]Timestamp + + // promoted list of persistence timestamp for each bucket. + persistenceTimestamps map[string][]PersistenceTimestamp + + // Index info for every created index. + indexeInfos map[uint64]IndexInfo + + // per index map of index topology + indexesTopology map[string]IndexTopology + + // per index map of ongoing rebalance. + rebalances map[string]Rebalance + + // list of projectors + projectors []Node + + // list of routers + routers [] Node + } +``` + +### failed, orphaned and outdated IndexManager + +1. when an IndexManager crashes, it shall be restarted by ns-server, join the + cluster, enter bootstrap state. +2. when a Index-Coordinator becomes unreachable to ns-server, it shall + restart itself, by joining the cluster after a timeout period and enter + bootstrap state. + +### client interfacing with Index-Coordinator + +1. a client must first get current Index-Coordinator from ns-server. If it + cannot get one, it must retry or fail. +2. once network address of Index-Coordinator is obtained from ns-server, client + can post update request to Index-Coordinator. +3. Index-Coordinator should create a new transaction in ns-server and then get + the current list of Index-Coordinator-Replicas from ns-server. + * if ns-server is unreachable or doesn't respond, return error to client. +5. Index-Coordinator goes through a 2-phase commit with its replicas and return + success or failure to client. + +### two-phase commit for co-ordination failures + +**false positive**, is a scenario when client thinks that an update request +succeeded but the system is not yet updated, due to failures. + +**false-negative**, is a scenario when client thinks that an update request has +failed but the system has applied the update into StateContext. + +Meanwhile, when system is going through a rebalance or a kv-rollback or +executing a new DDL statement, Index-Coordinator can crash. But this situation +should not put the system in in-consistent state. To achieve this, we propose a +solution with the help of ns-server. + +* Index-Coordinator and its replica shall maintain a persisted copy of its + local StateContext. +* ns-server will act as commit-point-site. +* for every update, Index-Coordinator will increment the CAS and create a + transaction tuple {CAS, status} with ns-server. +* status will be "initial" when it is created and later move to either "commit" + or "rollback". +* for every update accepted by Index-Coordinator it shall create a new copy of + StateContext, persist the new copy locally as log and push the copy to its + replicas. + * if one of the replica fails, Index-Coordinator will post "rollback" status + to ns-server for this transaction and issue a rollback to each of the + replica. + * upon rollback, Index-Coordinator and its replica will delete the local + StateContext log and return failure. +* when all the replicas accept the new copy of StateContext and persist it + locally as a log, Index-Coordinator will post "commit" status to + ns-server for this transaction and issue commit to each of the replica. +* during commit, local log will be switched as the new StateContext, and the + log file will be deleted. + +whenever a master or replica fails it will always go through a bootstrap phase +during which it shall detect a log of StateContext, consult with ns-server +for the corresponding transaction's status. If status is "commit" it will switch +the log as new StateContext, otherwise it shall delete the log and retain the +current StateContext. + +* once ns-server creates a new transaction entry, it should not accept a new + IndexManager into the cluster until the transaction moves to "commit" or + "rollback" status. +* during a master election ns-server shall pick a master only from the set of + IndexManagers that took part in the last-transaction. +* ns-server should maintain a rolling log of transaction status. +### bootstrap state for IndexManager + +* all IndexManagers, when they start afresh, will be in bootstrap State. +* from bootstrap State, IndexManager can either become Index-Coordinator or + Index-Replica. +* while a node is in bootstrap State it will wait for Index-Coordinator's + connectionAddress from ns-server or poll from ns-server. +* if connectionAddress is same as the IndexManager instance, it will become the + new Index-Coordinator. +* otherwise IndexManager will become a Replica to Index-Coordinator. + +* **recovery from failure**, before getting the new Index-Coordinator's +connectionAddress IndexManager will check of un-commit transaction by checking +for StateContext's log. + * If found, it will consult ns-server for transaction's status. + * if status is "commit", it will switch the log as new StateContext. + * else discard the log. + +* IndexManager before leaving bootstrap state will restore the StateContext + from its local persistence. + +#### Index-Coordinator initialization + +### normal state for Index-Coordinator + +* restored StateContext will be used as the master copy. Optionally + Index-Coordinator can consult its replica for latest StateContext based on + CAS value. +* StateContext can be modified only by the master node. It is expected that + other components within the system should somehow learn the current master + (maybe from ns-server) and use master's API to modify StateContext. +* upon receiving a new update to StateContext will use 2-phase commit to + replicate the update to its replica. + +#### Index-Coordinator publishing persistence timestamp + +Index-Coordinator will maintain a currentTimestamp vector for each bucket in +its StateContext, which will be updated using SYNC message received from +projector (projector will periodically send SYNC message for every vbucket in +a bucket). The sync message will contain the latest sequence-number for the +vbucket. + +Index-Coordinator will periodically recieve message from every +local-indexer-node. Based on HWHeartbeat metrics and/or query requirements, +Index-Coordinator will promote the currentTimestamp into a +persistence-timestamp and publish it to all index-nodes hosting a index for +that bucket. + +If, + * Index-Coordinator crashes while publishing the persistence timestamp, + * local-indexer-node did not receive the persistence-timestamp, + * local-indexer node received the timestamp but could not create a + snapshot due to a crash, + * compaction kicks in, +then there is no gaurantee that local-indexer-nodes hosting an index will have +identical snapshots. + +Whenever local-indexer node goes through a restart, it can fetch a log of +persistence-timestamp uptil the latest one from Index-Coordinator. + +##### algorithm to compute persistence timestamp + +Algorithm takes following as inputs. + +- per bucket HighWatermark-timestamp from each of the local-indexer-node. +- available free size in local-indexer's mutation-queue. + +1. For each vbucket, compute the mean seqNo +2. Use the mean seqNo to create a persistence timestamp +3. If heartbeat messages indicate that the faster indexer's mutation queue is + growing rapidly, it is possible to use a seqNo that matches that fast indexer + closer +4. If the local indexer that has not sent heartbeat messages within a certain + time, skip the local indexer, or consult the cluster mgr on the indexer + availability. +5. If the new persistent timestamp is the less than equal to the last one, do + nothing. + +**relevant data structur** + +```go + type PersistenceTimestamp struct { + ts Timestamp + stability bool // whether this timestamp is also treated as stability timestamp + } +``` + +##### /index-coordinator/hwtimestamp + +supported by Index-Coordinator + +request: + + { "command": HWheartbeat, + "payload": , + } + +response: + + { "cas": , + "status": ... + } + +##### /index-coordinator/logPersistenceTimestamps + +supported by Index-Coordinator + +request: + + { "command": PersistenceTimestampLog, + "lastPersistenceTimestamp": Timestamp, + } + +response: + + { "cas": , + "persistenceTimestamps": []PersistenceTimestamp + "status": ... + } + +##### /local-indexer-node/persistenceTimestamp + +supported by local-indexer-node + +request: + + { "cas": , + "command": NewPersistenceTimestamp, + "payload": , + } + +response: + + { "status": ... + } + +### Index-Coordinator replica + +* when a node move to replica State, it will first fetch the latest + StateContext from the current master and persist as the local StateContext. +* if replica's StateContext is newer than the master's StateContext + (detectable by comparing the CAS value), then latest mutations on the + replica will be lost. +* replica can receive updates to StateContext only from master, if it receives + from other components in the system, it will respond with error. +* upon receiving a new update to StateContext from master, replica will + update its StateContext and persist the StateContext on durable media. +* in case if replica is unable to communicate with the master and comes back + alive while rest of the system have moved ahead, it shall go through + bootstrap state, + * due to hearbeat failures + * by detecting the CAS value in subsequent updates from master + +### rollback + + * only master node can move to rollback mode. + + TBD + +## Data structure + +**cluster data structure** + +```go + type Cluster struct { + masterAddr string // connection address to master + replicas []string // list of connection address to replica nodes + nodes []string // list of connection address to indexer-nodes + } +``` + +data structure is transient and maintains the current state of the +secondary-index cluster + +**StateContext** + + type Rebalance struct { + topology IndexTopology + } + + type HWHeartbeat struct { + indexid uint64 + bucket string + hw Timestamp + lastPersistence uint64 // hash of Timestamp + lastStability uint64 // hash of Timestamp + mutationQueue uint64 + } +``` + +### IndexManager APIs + +#### /cluster/heartbeat +request: + + { "current_term": } + +response: + + { "current_term": } + +To be called by ns-server. Node will respond back with SUCCESS irrespective of +role or state. + +* if replica node does not respond back, it will be removed from active list. +* if master does not respond back, then ns-server can start a new + master-election. + +#### /cluster/bootstrap + +To be called by ns-server, ns-server is expected to make this request during +master election. + +* replica will cancel all outstanding updates and move to `bootstrap` state. +* master will cancel all outstanding updates and move to `bootstrap` state. + +#### /cluster/newmaster +request: + + { "current_term": } + +response: + + { "current_term": + "status": ... + } + +Once master election is completed ns-server will post the new master and +election-term to the elected master and each of its new replica. After this, +IndexManager node shall enter into `master` or `replica` state. + +### Index-Coordinator APIs + +#### /cluster/index +request: + + { "current_term": , + "command": CreateIndex, + "indexinfo": {}, + } + +response: + + { "current_term": + "status": ... + "indexinfo": {}, + } + +Create a new index specified by `indexinfo` field in request body. `indexinfo` +property is same as defined by the `IndexInfo` structure above, except that +`id` field will be generated by the master IndexManager and the same +`indexinfo` structure will be sent back as response. + +#### /cluster/index +request: + + { "current_term": , + "command": DropIndex, + "indexid": , + } + +response: + + { "current_term": + "status": ... + } + +Drop all index listed in `indexid` field. + +#### /cluster/index +request: + + { "current_term": , + "command": ListIndex, + "indexids": , + } + +response: + + { "current_term": , + "status": ... + "indexinfo": , + } + +List index meta-data structures identified by `indexids` in request body. If +it is empty, list all active indexes. + +### ns-server API requirements: + +#### GetIndexCoordinator() + +Returns connection address for current master. If no master is currently elected +then return empty string. + +#### GetIndexCoordinatorReplicas() + +Returns list of connection address for all active replicas in the system. + +#### GetIndexerNodes() + +Returns list of connection address for all active local-indexer-nodes in the system. + +#### Join() + +For a node to join the cluster + +#### Leave() + +For a node to leave the cluster + +## Appendix A: replication and fault-tolerance. + +To avoid single-point-of-failure in the system we need to have multiple +nodes that are continuously synchronized with each other. So that in case of a +a master failure, one of the replica can be promoted to master. In this +section we explore the challenges involved in synchronous replication of data +from master to its replica nodes. + +### full-replication + +Full replication means, for every update master will copy the entire data +structure to its replica. + +false-positive is possible when master crashes immediately and a new node, +that did not participate in the previous update, becomes a new master. This +can be avoided if master locks ns-server to prevent any new nodes from joining +the cluster until the update is completed. + +false-negative is possible when master crashes before updating all replicas +and one of the updated replica is elected as new master. + +Q: +1) When do the index manager needs to subscribe/unsubcribe directly? + Should the router handles the change the subscriber based on the topology? diff --git a/secondary/docs/design/markdown/indexer.md b/secondary/docs/design/markdown/indexer.md index 23dc3183a..47852b6bc 100644 --- a/secondary/docs/design/markdown/indexer.md +++ b/secondary/docs/design/markdown/indexer.md @@ -1 +1,13 @@ -##Indexer +## Indexer + +Indexer is a node in secondary-index cluster that will host IndexManager, +Indexer process. + +### Scope of local indexer + +1. run one or more instance of storage engine to persist projected key-versions. +2. manage index partitions and slices for each index. +3. independent restart path +4. Manage Storage Engine + * Control compaction frequency + * Snapshot Creation/Deletion diff --git a/secondary/docs/design/markdown/projector.md b/secondary/docs/design/markdown/projector.md index 62698d70e..c0ddcb93d 100644 --- a/secondary/docs/design/markdown/projector.md +++ b/secondary/docs/design/markdown/projector.md @@ -1 +1,60 @@ -##Projector +## Projector + +* projector is a stateless component in the system. That is, it is not backed by + any persistent storage. +* it fetches all relevant information form Index-Coordinator. +* connects with KV cluster for UPR mutations. +* evaluates each documents using index expression, if document belongs to the + bucket. +* creates projected key-versions and sends them to router component. +* when a new index is created or old index deleted, Index-Coordinator will post + the information to all projectors. + +## projector bootstrap + +* projector will get list of all active index-info from Index-Coordinator. +* projector will spawn a thread, referred to per-bucket-thread, for each + bucket that has an index defined. +* per-bucket-thread will, + * get a subset of vbuckets for which it had to start UPR streams. + * get a list of subscribers for each of these subsets who would like to + receive the key-version. + * open a UPR connection with one or more kv-master-nodes and start a stream + for each vbucket in its subset. + * apply expressions from each index defined for this bucket and generate + corresponding key-version. + * publish the key-version to subscribers. + +## changes to index DDLs + +Changes in index DDLs can affect projectors, + +* in case CREATE index DDL, + * projectors may have to spawn or kill a per-bucket thread. + * projectors may want the new index definition and its expression to generate + key-versions for the new index +* in case of DROP index DDL, + * projects may have kill a per-bucket thread. + * projects may want to delete index definition and its expression from its + local copy to stop generating key-versions for the delete index. +* sometimes to balance load, Index-Coordinator might change the subscriber + list who receive index key-versions. + +**relevant data structures** + +```go + type ProjectorState struct { + listenAddr string // address to send/receive administration messages + kvAddr string // address to connect with KV cluster + indexInfos []IndexInfo + } + + type KeyVersion struct { + docid []byte // primary document id. + vbucket uint16 // vbucket in which document is located. + vbuuid uint64 // current required uint64 vbuuid = 4; + sequenceno uint64 // sequence number corresponding to this mutation + indexid uint64 + keys []byte + } +``` diff --git a/secondary/docs/design/markdown/pubsub.md b/secondary/docs/design/markdown/pubsub.md new file mode 100644 index 000000000..349ca9646 --- /dev/null +++ b/secondary/docs/design/markdown/pubsub.md @@ -0,0 +1,90 @@ +## Publisher subscriber + +Publisher-subscriber is an event framework across the network. It is based on +`topics`, to which network subscriptions are allowed. The event framework is +actively managed by Index-Coordinator, and for every topic created or delete +and for every node subscribed or un-subscribed Index-Coordinator will manage +them as part of its StateContext. + +#### Topic + +Each topic is represented by following structure, + +```go + type Topic struct { + name string + subscribers []string + } +``` + +- `name` is represented in path format, and subscribe / un-subscribe APIs will + accept glob-pattern on name. +- `subscribers` is a list of connection string, where each connection string + is represented in **host:port** format. + +Subscribers should ensure that there is a thread/routine listening at the +specified port. Topic publishers will get the latest list of subscribers on +its topic, request a TCP connection with the subscriber and start pushing one +or more events to it. It is up to the publisher and the subscriber to agree +upon the connection, transport and protocol details. + +**Note:** In future we might extend this definition to make it easy for +third-party components to interplay with secondary indexing system + +### Special topic - TopicTopic + +TopicTopic is a special kind of topic than can selectively publish subscription +changes on any other topic. + +#### /topic/_path_ + +If a component want to be notified whenever a topic specified by `path` +changes, happens when a topic or a subscriber to topic is added or deleted, +process can subscribe to `/topic`, suffixed by the `path`. + +**Publisher: Index-Coordinator**, when ever Index-Coordinator makes a change +to an active topic or to the list of Topics, it will check for a topic name +`/topic/`, and push those changes to topictopic's subscribers. + +event format, +```json +{ + "topic": "/topic/", + "topic-name": , + "subscribers": +} +``` + +### Standard topics and publishers + +Following are standard collection of topics and its publishers. + +#### /indexinfos + +Subscribers will be notified when any index DDL changes in the StateContext. + +**Publisher: Index-Coordinator**, when ever Index-Coordinator makes a change +to an index or to the list of index-information, it will publish the following +event, + +```json +{ + "topic": "/indexinfos", + "indexinfo": +} +``` + +#### /indexinfos/_indexid_ + +Subscribers will be notified when index DDL for _indexid_ changes in the +StateContext. + +**Publisher: Index-Coordinator**, when ever Index-Coordinator makes a change +to index, `indexid`, it will publish the following event, + +```json +{ + "topic": "/indexinfos/", + "indexinfo": +} +``` diff --git a/secondary/docs/design/overview.md b/secondary/docs/design/overview.md index 4ac117354..3684322b0 100644 --- a/secondary/docs/design/overview.md +++ b/secondary/docs/design/overview.md @@ -1,39 +1,60 @@ -##Secondary Index Design Document +## Secondary Index Design Document ###Overview -This document describes the High Level Design for Secondary Indexes. It also describes the deployement options supported. +Secondary index provide the ability to query couchbase key-value data store. +In couchbase the primary data store is a key-value storage mainly backed by +main-memory and distributed across dozens of nodes. If you are new couchbase +or couchbase's secondary indexing solution, you may first want learn more about +the [terminologies](markdown/terminology.md) that we discuss here. + +This document describes the High Level Design for Secondary Indexes. It also +describes the deployment options supported. ###Components - __Projector__ - The projector is responsible for mapping mutations to a set of key version. The projector can reside within the master KV node in which the mutation is generated or it can reside in separate node. The projector receives mutations from ep-engine through UPR protocol. The projector sends the evaluated results to router. [Details.](markdown/projector.md) +The projector is responsible for mapping mutations to a set of key version. +The projector can reside within the master KV node in which the mutation is +generated or it can reside in separate node. The projector receives mutations +from ep-engine through UPR protocol. The projector sends the evaluated results +to router. [Details.](markdown/projector.md) - __Router__ - The router is responsible for sending key version to the index nodes. It relies on the index distribution/partitioning topology to determine the indexer which should receive the key version. The router resides in the same node as the projector. [Details.](markdown/router.md) - +The router is responsible for sending key version to the index nodes. It relies +on the index distribution/partitioning topology to determine the indexer which +should receive the key version. The router resides in the same node as the +projector. [Details.](markdown/router.md) + - __Index Manager__ - The index manager is responsible for receiving requests for indexing operations (creation, deletion, maintenance, scan/lookup). The Index Manager is located in the index node, which can be different from KV node. [Details.](markdown/index_manager.md) - +The index manager is responsible for receiving requests for indexing operations +(creation, deletion, maintenance, scan/lookup). The Index Manager is located in +the index node, which can be different from KV node. +[Details.](markdown/index_manager.md) + - __Indexer__ - The indexer processes the key versions received from router and provides persistence support for the index. Indexer also provides the interface for query client to run index Scans and does scatter/gather for queries. The indexer would reside in index node. [Details.](markdown/indexer.md) - -- __Query Catalog__ +The indexer processes the key versions received from router and provides +persistence support for the index. Indexer also provides the interface for +query client to run index Scans and does scatter/gather for queries. The indexer +would reside in index node. [Details.](markdown/indexer.md) - This component provides catalog implementation for the Query Server. This component resides in the same node Query Server is running and allows Query Server to perform Index DDL (Create, Drop) and Index Scan/Stats operations. +- __Query Catalog__ +This component provides catalog implementation for the Query Server. This +component resides in the same node Query Server is running and allows Query +Server to perform Index DDL (Create, Drop) and Index Scan/Stats operations. ###System Diagram - [KV-Index System Diagram](markdown/system.md) - Query-Index System Diagram - + ###Execution Flow * [Mutation Execution Flow](markdown/mutation.md) @@ -52,40 +73,43 @@ This document describes the High Level Design for Secondary Indexes. It also des - [Deployment Options](markdown/deployment.md) ###Partition Management -* Milestone1 will have Key-based partitioning support. +* Milestone-1 will have Key-based partitioning support. * [John's Doc for Partitioning](https://docs.google.com/document/d/1eF3rJ63iv1awnfLkAQLmVmILBdgD4Vzc0IsCpTxmXgY/edit) ###Communication Protocols * Projector and Ep-Engine Protocol - * [UPR protocol](https://github.com/couchbaselabs/cbupr/blob/master/index.md) will be used to talk to Ep-engine in KV. - + * [UPR protocol](https://github.com/couchbaselabs/cbupr/blob/master/index.md) will be used to talk to Ep-engine in KV. + * Router and Indexer Protocol * Query and Indexer Protocol * [Existing REST Based Protocol](https://docs.google.com/document/d/1j9D4ryOi1d5CNY5EkoRuU_fc5Q3i_QwIs3zU9uObbJY/edit) ###Storage Management -* Persistent Snapshot + +* Persistent Snapshot ###Cluster Management + * Master Election * Communication with ns_server -###Metadata Management +### Meta-data Management + * Metadata Replication * Metadata Recovery ###Recovery -* [Recovery Document](https://docs.google.com/document/d/1rNJSVs80TtvY0gpoebsBwzhqWRBJnieSuLTnxuDzUTQ/edit) + +* [Recovery Document](https://docs.google.com/document/d/1rNJSVs80TtvY0gpoebsBwzhqWRBJnieSuLTnxuDzUTQ/edit) ###Replication + * Replication Strategy * Failure Recovery ###Rebalance + * Rebalance Strategy * Failure Recovery -###Terminology - -* [Terminology Document](markdown/terminology.md) diff --git a/secondary/docs/development.md b/secondary/docs/development.md new file mode 100644 index 000000000..0137b523d --- /dev/null +++ b/secondary/docs/development.md @@ -0,0 +1,27 @@ +##Documentation + +- All documentation is done through text markup, preferably with markdown + syntax. + +- We encourage using diagrams where ever it is relevant or necessary. Few of + us use yed to draw diagrams and save it as `graphml` files. You can add + references to these images from within your markdown documents. + +- To enable spell checking in vim use this option + > :setlocal spell spelllang=en_us + + to learn more about spell-checker shortcuts. + > :help spell + +- It is convenient to use chrome plugins like `Markdown Preview` to visually + check document formatting before committing your changes to git. + After installing it from chrome-extensions, enable "Allow access to file URLs" + in Extensions (menu > Tools > Extensions). Then open the file in chrome to + see the visual effects of markdown. + +- If the markdown file is stored with `.md` file extension, it could get + interpreted a modula-2 file. To enable correct syntax highlighting refer to + your editor documentation. + +- **please try to use 80 column width for text documents**, although it is not a + strict rule let there be a valid reason for lines exceeding 80 columns.