概要
- 設定ファイルはDatabaseDescriptorによって解析されます。(デフォルト値がある項目については、そのデフォルト値も定義しています。)
- ThriftによってCassanda.java内にAPIのインターフェースが生成されています。実装はCassandraServerが行い、CassandraDaemonがそれらを結びつけます。
- CassandraServerがThriftのリクエストを内部処理に変換し、StorageProxyが実際にリクエストの処理を行います。CassandraServerはThrift用に結果を変換して返します。
- StorageServiceは内部的にCassandraDaemonに対応するようなものです。素のゴシッププロトコルを正確に内部状態に変換する役割を持ちます。
- AbstractReplicationStrategyが2つ目、3つ目、...のレプリカを受け取るノードをコントロールします。最初のレプリカは(TokenMetadata内の)トークンリングによって決められます。しかしそれ以降のレプリカに関してはいろいろなバリエーションをとることができます。RackUnaware は単にリング上の次のN-1番目のノードにレプリカを配置します。RackAware は2番目のレプリカを、最初のレプリカが配置されたのとは別のデータセンターにあるリング上の次のノードに配置します。そして残りのレプリカを最初のレプリカと同じように配置します。
- MessagingServiceが接続プールと適切なステージ(基本的にはスレッド化されたExecutorService)での内部コマンドの実行を制御します。ステージはStageManager内に定義されています。現時点では、読み込み、書き込み、そしてストリームのステージが有ります。(ストリーミングはノードの起動時やリング上での再配置時に、あるノードから別のノードへSSTableの大部分をコピーする場合に使用されます。) 内部コマンドはStorageServiceで定義されます。
registerVerbHandlers
を参照してください。
書き込み時の流れ
- StorageProxyがReplicationStrategyからキーのレプリカを保持するノードを取得し、それらにRowMutationメッセージを送信します。
- もしリング上でノードの位置が変更している場合、TokenMetadata内で"保留中のレンジ"は目的地に関連付けられ、それらに書き込まれます。
- もし書き込み対象のノードが落ちていて、残りのノードでConsistencyLevelの要求を満たすことが出来る場合、落ちているノードに対する書き込みは他のノードに送信されます。その際に、そのノードが復帰した際にそのキーに関連付けられたデータはレプリカノードに送信されるべきだというヘッダ("ヒント")をつけて送信します。この処理はHintedHandoffと呼ばれ、イベンチュアルコンシステンシーの"イベンチャル"に到達するまでの時間を減らします。HintedHandoffは*最適化*に過ぎません。より完全な一貫性の保持についてはAntiEntropyが責任をもちます。
- 受け取る側のノードでは、RowMutationVerbHandlerが書き込みをまず最初にCommitLog.javaに渡します。そして(Table.applyメソッド経由で)適切なColumnFamilyのMemtableへ書きこみます。
- Memtableが一杯になった場合、ソートされてColumnFamilyStore.switchMemtableの処理で非同期にSSTableとして書き出されます。
- SSTableがある程度存在する場合、ColumnFamilyStore.doFileCompaction処理によってマージされます。
- 古いSSTableを削除して新しいSSTableを作成している間に、書き込みや読み込みをブロックすることなくこの処理を安全に並列化するには注意が必要です。なぜなら単純なアプローチでは古いSSTableを削除する前にそれらを読み込みんでいるリーダーのすべてが読み込みを完了するのを待つ必要があります。(実際にファイルオープンが開始されたかを知ることはできないからです。もしまだ開始されておらず、ファイルの削除が先に実行された場合、読み込みはエラーになるでしょう。) Cassandraでは、古いSSTableを同期して削除しないアプローチをとることになりました。代わりに、ガベージコレクタにファントム参照オブジェクト(PhantomReference)を登録し、SSTableへの参照がなくなった時点で消去されるようにしました。(同時にファイルシステムへコンパクションの目印を書きこみます。こうしておくことでコンパクションが起こる前にサーバーが再起動してしまった場合でも、起動時に古いSSTableを消去できます。)
- 詳細はArchitectureCommitLog]と[ArchitectureCommitLogを参照してください。
読み取り時の流れ
- StorageProxyがReplicationStrategyからそのキーのレプリカを保持しているノードを取得し、それらに読み取りメッセージを送信します。
- メッセージはSliceFromReadCommand、SliceByNamesReadCommand、もしくはRangeSliceReadCommandです。
- データを保持しているノードでは、ReadVerbHandlerがColumnFamilyStore.getColumnFamilyを使用してデータを取得し、ReadResponseとして送り返します。
- SSTableReader.getPosition内でインデックスに対してバイナリサーチを行い、行を特定します。
- 1行の問い合わせでは、QueryFilterのサブクラスを使用して、MemtableとSSTableから探しているデータを取得します。Memtableからの読み込みは直接的です。SSTableからの読み取りはリクエストの種類に応じて多少異なります。
- カラムのスライスを読み込もうとしている場合、行レベルのカラムインデックスを用いて読み取りの開始場所を決定し、1度に1ブロックづつデシリアライズします。("ブロック"とはひとつのインデックスによってカバーされるカラムのグループです。)そうすることで、"逆の"場合に大部分をメモリに読み込む事なく処理できます。
- カラムのグループを名前で読み込もうとしている場合もカラムインデックスを用いて各カラムの位置を特定しますが、まず最初に行レベルのブルームフィルタをチェックし、そのファイルからの読み込みがそもそも必要かをチェックします。
- カラムの読み込みにはIteratorインターフェースが提供されています。このことにより、フィルタは読み込みが完了したらすぐに停止することができます。必要ないカラムを読み取ることはありません。
- 潜在的に複数のSSTableのバージョンからカラムをマージする必要があるため、読み込みイテレータはReducingIteratorを使用して組み合わされます。まだ組み合わされていないカラムのイテレータをインプットとして受け取り、組み合わされたバージョンをアウトプットとして生成します。
- Quorumレベルの読み込みが要求された場合、StorageProxyはノードの大部分が応答を返すのを待ちます。そして結果を返す前にそれらの応答がマッチするかを確認します。それ以外の場合では結果をすぐに返します。そしてStorageService.doConsistencyCheckを行い、バックグラウンドでレプリカの不整合をチェックします。これは"リードリペア(Read repair)"と呼ばれ、一貫性の保証を素早く行う手助けをします。
- 最適化のため、StorageProxyは一番近いレプリカへ実際のデータを問い合せます。他のレプリカはデータのハッシュ値を計算するためだけに問い合せます。
データの削除
ゴシッププロトコル
故障の検出
より理解を深めるために
- タスクを"ステージ"に分割して別々のスレッドプールを割り当てるアイディアはSEDA(PDF)の論文から頂きました。
- 「クラッシュオンリー(Crash-only)」な設計は、広範囲にわたって適用されている原則です。Valerie HensonのLWNにおける記事が入門には最適です。
- Cassandraの分散システムの仕組みは、AmazonのDynamo論文に記載されているものに密接に関連しています。Read repair、調整可能な一貫性レベル、Hinted Handoff、その他のコンセプトが議論されています。これはバックグラウンドの知識として必読のマテリアルです。 他にイベンチュアルコンシステンシーに関する記事も関連があります。
- Cassandraのディスク上のストレージモデルはBigtableの論文のセクション5.3と5.4にほぼ基づいています。
- FacebookのCassandraチームがLADIS09でCassandraの論文(PDF)を発表しました。いまでは、ZooKeeperとの統合部分が主な違いとなっています。
https://c.statcounter.com/9397521/0/fe557aad/1/|stats