Initial import from separate projects

This commit is contained in:
Christian Kamm 2021-11-01 10:34:25 +01:00
commit 11ba681f9c
15 changed files with 7445 additions and 0 deletions

1
.gitignore vendored Normal file
View File

@ -0,0 +1 @@
/target

6049
Cargo.lock generated Normal file

File diff suppressed because it is too large Load Diff

20
Cargo.toml Normal file
View File

@ -0,0 +1,20 @@
[workspace]
members = [
"accountsdb-plugin-grpc",
"connector-lib",
]
[patch.crates-io]
# for gzip encoded responses
jsonrpc-core-client = { path = "../jsonrpc-dep/core-client" }
#solana-accountsdb-plugin-interface = { path = "../solana-accountsdb/accountsdb-plugin-interface" }
#solana-logger = { path = "../solana-accountsdb/logger" }
#solana-metrics = { path = "../solana-accountsdb/metrics" }
#solana-sdk = { path = "../solana-accountsdb/sdk" }
#solana-program = { path = "../solana-accountsdb/sdk/program" }
#anchor-lang = { path = "../anchor/lang" } # armani/solana branch for 1.8
#mango = { path = "../mango-v3-dep/program" }

View File

@ -0,0 +1,45 @@
[package]
name = "solana-accountsdb-connector-plugin-grpc"
version = "0.1.0"
authors = ["Christian Kamm <ckamm@delightful-solutions.de>"]
edition = "2018"
[lib]
crate-type = ["cdylib", "rlib"]
[[bin]]
name = "test-server"
path = "src/test_server.rs"
[dependencies]
bs58 = "0.4.0"
chrono = { version = "0.4.11", features = ["serde"] }
crossbeam-channel = "0.5"
log = "0.4.14"
serde = "1.0.130"
serde_derive = "1.0.103"
serde_json = "1.0.67"
solana-accountsdb-plugin-interface = "=1.8.2"
solana-logger = "=1.8.2"
solana-metrics = "=1.8.2"
solana-sdk = "=1.8.2"
thiserror = "1.0.21"
tonic = "0.6"
prost = "0.9"
futures-core = "0.3"
futures-util = "0.3"
tokio = { version = "1.0", features = ["rt-multi-thread", "macros", "sync", "time"] }
tokio-stream = "0.1"
async-stream = "0.2"
rand = "0.7"
[build-dependencies]
tonic-build = "0.6"
[package.metadata.docs.rs]
targets = ["x86_64-unknown-linux-gnu"]

View File

@ -0,0 +1,4 @@
fn main() {
tonic_build::compile_protos("../proto/accountsdb.proto")
.unwrap_or_else(|e| panic!("Failed to compile protos {:?}", e));
}

View File

@ -0,0 +1,69 @@
use {log::*, std::collections::HashSet};
#[derive(Debug)]
pub(crate) struct AccountsSelector {
pub accounts: HashSet<Vec<u8>>,
pub owners: HashSet<Vec<u8>>,
pub select_all_accounts: bool,
}
impl AccountsSelector {
pub fn default() -> Self {
AccountsSelector {
accounts: HashSet::default(),
owners: HashSet::default(),
select_all_accounts: true,
}
}
pub fn new(accounts: &[String], owners: &[String]) -> Self {
info!(
"Creating AccountsSelector from accounts: {:?}, owners: {:?}",
accounts, owners
);
let select_all_accounts = accounts.iter().any(|key| key == "*");
if select_all_accounts {
return AccountsSelector {
accounts: HashSet::default(),
owners: HashSet::default(),
select_all_accounts,
};
}
let accounts = accounts
.iter()
.map(|key| bs58::decode(key).into_vec().unwrap())
.collect();
let owners = owners
.iter()
.map(|key| bs58::decode(key).into_vec().unwrap())
.collect();
AccountsSelector {
accounts,
owners,
select_all_accounts,
}
}
pub fn is_account_selected(&self, account: &[u8], owner: &[u8]) -> bool {
self.select_all_accounts || self.accounts.contains(account) || self.owners.contains(owner)
}
}
#[cfg(test)]
pub(crate) mod tests {
use super::*;
#[test]
fn test_create_accounts_selector() {
AccountsSelector::new(
&["9xQeWvG816bUx9EPjHmaT23yvVM2ZWbrrpZb9PusVFin".to_string()],
&[],
);
AccountsSelector::new(
&[],
&["9xQeWvG816bUx9EPjHmaT23yvVM2ZWbrrpZb9PusVFin".to_string()],
);
}
}

View File

@ -0,0 +1,306 @@
use {
crate::accounts_selector::AccountsSelector,
accountsdb_proto::{
slot_update::Status as SlotUpdateStatus, update::UpdateOneof, AccountWrite, SlotUpdate,
SubscribeRequest, Update,
},
bs58,
futures_util::FutureExt,
log::*,
serde_derive::{Deserialize, Serialize},
serde_json,
solana_accountsdb_plugin_interface::accountsdb_plugin_interface::{
AccountsDbPlugin, AccountsDbPluginError, ReplicaAccountInfoVersions,
Result as PluginResult, SlotStatus,
},
std::sync::Mutex,
std::{fs::File, io::Read},
thiserror::Error,
tokio::sync::{broadcast, mpsc, oneshot},
tonic::transport::Server,
};
pub mod accountsdb_proto {
tonic::include_proto!("accountsdb");
}
pub mod accountsdb_service {
use super::*;
use {
accountsdb_proto::accounts_db_server::{AccountsDb, AccountsDbServer},
tokio_stream::wrappers::ReceiverStream,
tonic::{Code, Request, Response, Status},
};
#[derive(Debug)]
pub struct Service {
pub sender: broadcast::Sender<Update>,
}
impl Service {
pub fn new() -> Self {
let (tx, _) = broadcast::channel(100);
Self { sender: tx }
}
}
#[tonic::async_trait]
impl AccountsDb for Service {
type SubscribeStream = ReceiverStream<Result<Update, Status>>;
async fn subscribe(
&self,
_request: Request<SubscribeRequest>,
) -> Result<Response<Self::SubscribeStream>, Status> {
println!("new client");
let (tx, rx) = mpsc::channel(100);
let mut broadcast_rx = self.sender.subscribe();
tokio::spawn(async move {
let mut exit = false;
while !exit {
let fwd = broadcast_rx.recv().await.map_err(|err| {
// TODO: Deal with lag! though maybe just close if RecvError::Lagged happens
warn!("error while receiving message to be broadcast: {:?}", err);
exit = true;
Status::new(Code::Internal, err.to_string())
});
if let Err(err) = tx.send(fwd).await {
warn!(
"error while sending message to subscriber stream: {:?}",
err
);
}
}
});
Ok(Response::new(ReceiverStream::new(rx)))
}
}
}
#[derive(Default)]
pub struct AccountsDbPluginGrpc {
server_broadcast: Option<broadcast::Sender<Update>>,
server_exit_sender: Option<oneshot::Sender<()>>,
accounts_selector: Option<AccountsSelector>,
}
impl std::fmt::Debug for AccountsDbPluginGrpc {
fn fmt(&self, _: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
Ok(())
}
}
#[derive(Clone, Debug, PartialEq, Serialize, Deserialize)]
pub struct AccountsDbPluginGrpcConfig {
pub bind_string: String,
}
#[derive(Error, Debug)]
pub enum AccountsDbPluginGrpcError {}
impl AccountsDbPluginGrpc {
fn broadcast(&self, update: UpdateOneof) {
if let Some(sender) = &self.server_broadcast {
// Don't care about the error if there are no receivers.
let _ = sender.send(Update {
update_oneof: Some(update),
});
}
}
}
impl AccountsDbPlugin for AccountsDbPluginGrpc {
fn name(&self) -> &'static str {
"AccountsDbPluginGrpc"
}
fn on_load(&mut self, config_file: &str) -> PluginResult<()> {
solana_logger::setup_with_default("info");
info!(
"Loading plugin {:?} from config_file {:?}",
self.name(),
config_file
);
let mut file = File::open(config_file)?;
let mut contents = String::new();
file.read_to_string(&mut contents)?;
let result: serde_json::Value = serde_json::from_str(&contents).unwrap();
self.accounts_selector = Some(Self::create_accounts_selector_from_config(&result));
let config: AccountsDbPluginGrpcConfig =
serde_json::from_str(&contents).map_err(|err| {
AccountsDbPluginError::ConfigFileReadError {
msg: format!(
"The config file is not in the JSON format expected: {:?}",
err
),
}
})?;
let addr = config.bind_string.parse().map_err(|err| {
AccountsDbPluginError::ConfigFileReadError {
msg: format!("Error parsing the bind_string {:?}", err),
}
})?;
let service = accountsdb_service::Service::new();
let (exit_sender, exit_receiver) = oneshot::channel::<()>();
self.server_exit_sender = Some(exit_sender);
self.server_broadcast = Some(service.sender.clone());
let server = accountsdb_proto::accounts_db_server::AccountsDbServer::new(service);
tokio::spawn(
Server::builder()
.add_service(server)
.serve_with_shutdown(addr, exit_receiver.map(drop)),
);
Ok(())
}
fn on_unload(&mut self) {
info!("Unloading plugin: {:?}", self.name());
if let Some(sender) = self.server_exit_sender.take() {
sender
.send(())
.expect("sending grpc server termination should succeed");
}
}
fn update_account(
&mut self,
account: ReplicaAccountInfoVersions,
slot: u64,
is_startup: bool,
) -> PluginResult<()> {
match account {
ReplicaAccountInfoVersions::V0_0_1(account) => {
if let Some(accounts_selector) = &self.accounts_selector {
if !accounts_selector.is_account_selected(account.pubkey, account.owner) {
return Ok(());
}
} else {
return Ok(());
}
debug!(
"Updating account {:?} with owner {:?} at slot {:?} using account selector {:?}",
bs58::encode(account.pubkey).into_string(),
bs58::encode(account.owner).into_string(),
slot,
self.accounts_selector.as_ref().unwrap()
);
// TODO: send the update to all connected streams
self.broadcast(UpdateOneof::AccountWrite(AccountWrite {
slot,
is_startup,
write_version: account.write_version,
pubkey: account.pubkey.to_vec(),
lamports: account.lamports,
owner: account.owner.to_vec(),
executable: account.executable,
rent_epoch: account.rent_epoch,
data: account.data.to_vec(),
}));
}
}
Ok(())
}
fn update_slot_status(
&mut self,
slot: u64,
parent: Option<u64>,
status: SlotStatus,
) -> PluginResult<()> {
info!("Updating slot {:?} at with status {:?}", slot, status);
let status = match status {
SlotStatus::Processed => SlotUpdateStatus::Processed,
SlotStatus::Confirmed => SlotUpdateStatus::Confirmed,
SlotStatus::Rooted => SlotUpdateStatus::Rooted,
};
self.broadcast(UpdateOneof::SlotUpdate(SlotUpdate {
slot,
parent,
status: status as i32,
}));
Ok(())
}
fn notify_end_of_startup(&mut self) -> PluginResult<()> {
Ok(())
}
}
impl AccountsDbPluginGrpc {
fn create_accounts_selector_from_config(config: &serde_json::Value) -> AccountsSelector {
let accounts_selector = &config["accounts_selector"];
if accounts_selector.is_null() {
AccountsSelector::default()
} else {
let accounts = &accounts_selector["accounts"];
let accounts: Vec<String> = if accounts.is_array() {
accounts
.as_array()
.unwrap()
.iter()
.map(|val| val.as_str().unwrap().to_string())
.collect()
} else {
Vec::default()
};
let owners = &accounts_selector["owners"];
let owners: Vec<String> = if owners.is_array() {
owners
.as_array()
.unwrap()
.iter()
.map(|val| val.as_str().unwrap().to_string())
.collect()
} else {
Vec::default()
};
AccountsSelector::new(&accounts, &owners)
}
}
pub fn new() -> Self {
AccountsDbPluginGrpc {
server_broadcast: None,
server_exit_sender: None,
accounts_selector: None,
}
}
}
#[no_mangle]
#[allow(improper_ctypes_definitions)]
/// # Safety
///
/// This function returns the AccountsDbPluginGrpc pointer as trait AccountsDbPlugin.
pub unsafe extern "C" fn _create_plugin() -> *mut dyn AccountsDbPlugin {
let plugin = AccountsDbPluginGrpc::new();
let plugin: Box<dyn AccountsDbPlugin> = Box::new(plugin);
Box::into_raw(plugin)
}
#[cfg(test)]
pub(crate) mod tests {
use {super::*, serde_json};
#[test]
fn test_accounts_selector_from_config() {
let config = "{\"accounts_selector\" : { \
\"owners\" : [\"9xQeWvG816bUx9EPjHmaT23yvVM2ZWbrrpZb9PusVFin\"] \
}}";
let config: serde_json::Value = serde_json::from_str(config).unwrap();
AccountsDbPluginGrpc::create_accounts_selector_from_config(&config);
}
}

View File

@ -0,0 +1,2 @@
pub mod accounts_selector;
pub mod accountsdb_plugin_grpc;

View File

@ -0,0 +1,86 @@
use futures_core::Stream;
use std::pin::Pin;
use std::sync::Arc;
use tokio::sync::{broadcast, mpsc};
use tokio_stream::wrappers::ReceiverStream;
use tonic::transport::Server;
use tonic::{Request, Response, Status};
pub mod accountsdb_proto {
tonic::include_proto!("accountsdb");
}
use accountsdb_proto::{update::UpdateOneof, AccountWrite, SlotUpdate, SubscribeRequest, Update};
pub mod accountsdb_service {
use super::*;
use {
accountsdb_proto::accounts_db_server::{AccountsDb, AccountsDbServer},
tokio_stream::wrappers::ReceiverStream,
tonic::{Request, Response, Status},
};
#[derive(Debug)]
pub struct Service {
pub sender: broadcast::Sender<Update>,
}
impl Service {
pub fn new() -> Self {
let (tx, _) = broadcast::channel(100);
Self { sender: tx }
}
}
#[tonic::async_trait]
impl AccountsDb for Service {
type SubscribeStream = ReceiverStream<Result<Update, Status>>;
async fn subscribe(
&self,
_request: Request<SubscribeRequest>,
) -> Result<Response<Self::SubscribeStream>, Status> {
println!("new client");
let (tx, rx) = mpsc::channel(100);
let mut broadcast_rx = self.sender.subscribe();
tokio::spawn(async move {
loop {
// TODO: Deal with lag! maybe just close if RecvError::Lagged happens
let msg = broadcast_rx.recv().await.unwrap();
tx.send(Ok(msg)).await.unwrap();
}
});
Ok(Response::new(ReceiverStream::new(rx)))
}
}
}
#[tokio::main]
async fn main() -> Result<(), Box<dyn std::error::Error>> {
let addr = "[::1]:10000".parse().unwrap();
let service = accountsdb_service::Service::new();
let sender = service.sender.clone();
let svc = accountsdb_proto::accounts_db_server::AccountsDbServer::new(service);
tokio::spawn(async move {
loop {
if sender.receiver_count() > 0 {
println!("sending...");
sender
.send(Update {
update_oneof: Some(UpdateOneof::SlotUpdate(SlotUpdate {
slot: 0,
parent: None,
status: 0,
})),
})
.unwrap();
}
tokio::time::sleep(std::time::Duration::from_secs(1)).await;
}
});
Server::builder().add_service(svc).serve(addr).await?;
Ok(())
}

40
connector-lib/Cargo.toml Normal file
View File

@ -0,0 +1,40 @@
[package]
name = "solana-accountsdb-connector-lib"
version = "0.1.0"
authors = ["Christian Kamm <ckamm@delightful-solutions.de>"]
edition = "2018"
[dependencies]
jsonrpc-core = "18.0.0"
jsonrpc-derive = "18.0.0"
jsonrpc-http-server = "18.0.0"
jsonrpc-core-client = { version = "18.0.0", features = ["ws", "http"] }
solana-rpc = "=1.8.2"
solana-client = "=1.8.2"
solana-account-decoder = "1.8.2"
solana-sdk = "^1.7.10"
hyper = { version = "0.14", features = ["client", "http1"] }
hyper-tls = { version = "0.5" }
tokio = { version = "1", features = ["full"] }
tokio-postgres = "0.7.4"
tokio-stream = "0.1"
postgres_query = { git = "https://github.com/nolanderc/rust-postgres-query", rev = "b4422051c8a31fbba4a35f88004c1cefb1878dd5" }
tonic = "0.6"
fixed = { version = "=1.9.0", features = ["serde"] }
serde = "^1.0.118"
bs58 = "0.3.1"
futures = "0.3.17"
log = "0.4"
prost = "0.9"
futures-core = "0.3"
futures-util = "0.3"
async-stream = "0.2"
rand = "0.7"
anyhow = "1.0"
[build-dependencies]
tonic-build = "0.6"

4
connector-lib/build.rs Normal file
View File

@ -0,0 +1,4 @@
fn main() {
tonic_build::compile_protos("../proto/accountsdb.proto")
.unwrap_or_else(|e| panic!("Failed to compile protos {:?}", e));
}

743
connector-lib/src/main.rs Normal file
View File

@ -0,0 +1,743 @@
use jsonrpc_core::futures::StreamExt;
use jsonrpc_core_client::transports::{http, ws};
use jsonrpc_derive::rpc;
use jsonrpc_http_server::ServerBuilder;
use solana_account_decoder::UiAccountEncoding;
use solana_client::{
rpc_config::{RpcAccountInfoConfig, RpcProgramAccountsConfig},
rpc_filter::RpcFilterType,
rpc_response::{Response, RpcKeyedAccount},
};
use solana_rpc::{rpc::rpc_full::FullClient, rpc::OptionalContext, rpc_pubsub::RpcSolPubSubClient};
use solana_sdk::{account::Account, commitment_config::CommitmentConfig, pubkey::Pubkey};
//use solana_program::account_info::AccountInfo;
//use mango::state::{MangoAccount, MangoCache, MangoGroup};
use fixed::types::I80F48;
use futures::FutureExt;
use log::{error, info, trace, warn};
use postgres_query::{query, FromSqlRow};
use serde::{Deserialize, Serialize};
use std::{
cmp::max,
collections::{HashMap, HashSet, VecDeque},
mem::size_of,
ops::Deref,
rc::Rc,
str::FromStr,
sync::{mpsc, Arc, RwLock},
thread::sleep,
time::{Duration, Instant},
};
use tokio::{runtime::Runtime, time};
pub mod accountsdb_proto {
tonic::include_proto!("accountsdb");
}
use accountsdb_proto::accounts_db_client::AccountsDbClient;
struct SimpleLogger;
impl log::Log for SimpleLogger {
fn enabled(&self, metadata: &log::Metadata) -> bool {
metadata.level() <= log::Level::Info
}
fn log(&self, record: &log::Record) {
if self.enabled(record.metadata()) {
println!("{} - {}", record.level(), record.args());
}
}
fn flush(&self) {}
}
static LOGGER: SimpleLogger = SimpleLogger;
/*
pub struct MangoData {
group: Option<MangoGroup>,
cache: Option<MangoCache>,
}
impl MangoData {
fn new() -> Self {
Self {
group: None,
cache: None,
}
}
}
//
// updating mango data
//
async fn get_account(client: &FullClient, pubkey: String) -> Option<Account> {
client
.get_account_info(
pubkey,
Some(RpcAccountInfoConfig {
encoding: Some(UiAccountEncoding::Base64),
commitment: Some(CommitmentConfig::processed()),
data_slice: None,
}),
)
.await
.ok()?
.value?
.decode()
}
async fn get_mango_group(
client: &FullClient,
group_key: &Pubkey,
program_id: &Pubkey,
) -> Option<MangoGroup> {
let mut account = get_account(client, group_key.to_string()).await?;
let accountInfo = AccountInfo::from((group_key, &mut account));
let group = MangoGroup::load_checked(&accountInfo, program_id)
.ok()?
.deref()
.clone();
Some(group)
}
async fn get_mango_cache(
client: &FullClient,
group: &MangoGroup,
program_id: &Pubkey,
) -> Option<MangoCache> {
let mut account = get_account(client, group.mango_cache.to_string()).await?;
let accountInfo = AccountInfo::from((&group.mango_cache, &mut account));
let cache = MangoCache::load_checked(&accountInfo, program_id, group)
.ok()?
.deref()
.clone();
Some(cache)
}
async fn update_mango_data(mango_data: Arc<RwLock<MangoData>>) {
let program_id =
Pubkey::from_str("mv3ekLzLbnVPNxjSKvqBpU3ZeZXPQdEC3bp5MDEBG68").expect("valid pubkey");
let mango_group_address =
Pubkey::from_str("98pjRuQjK3qA6gXts96PqZT4Ze5QmnCmt3QYjhbUSPue").expect("valid pubkey");
let rpc_http_url = "";
let rpc_client = http::connect_with_options::<FullClient>(&rpc_http_url, true)
.await
.unwrap();
loop {
let group = get_mango_group(&rpc_client, &mango_group_address, &program_id)
.await
.unwrap();
let cache = get_mango_cache(&rpc_client, &group, &program_id)
.await
.unwrap();
{
let mut writer = mango_data.write().unwrap();
writer.group = Some(group);
writer.cache = Some(cache);
}
tokio::time::sleep(Duration::from_secs(10)).await;
}
}*/
//
// main etc.
//
enum WebsocketMessage {
SingleUpdate(Response<RpcKeyedAccount>),
SnapshotUpdate(Response<Vec<RpcKeyedAccount>>),
SlotUpdate(Arc<solana_client::rpc_response::SlotUpdate>),
}
trait AnyhowWrap {
type Value;
fn map_err_anyhow(self) -> anyhow::Result<Self::Value>;
}
impl<T, E: std::fmt::Debug> AnyhowWrap for Result<T, E> {
type Value = T;
fn map_err_anyhow(self) -> anyhow::Result<Self::Value> {
self.map_err(|err| anyhow::anyhow!("{:?}", err))
}
}
async fn feed_data(sender: mpsc::Sender<WebsocketMessage>) -> Result<(), anyhow::Error> {
let rpc_pubsub_url = "";
let rpc_http_url = "";
let program_id = Pubkey::from_str("mv3ekLzLbnVPNxjSKvqBpU3ZeZXPQdEC3bp5MDEBG68")?;
let mango_group_address = Pubkey::from_str("98pjRuQjK3qA6gXts96PqZT4Ze5QmnCmt3QYjhbUSPue")?;
let snapshot_duration = Duration::from_secs(300);
let connect = ws::try_connect::<RpcSolPubSubClient>(&rpc_pubsub_url).map_err_anyhow()?;
let client = connect.await.map_err_anyhow()?;
let rpc_client = http::connect_with_options::<FullClient>(&rpc_http_url, true)
.await
.map_err_anyhow()?;
let account_info_config = RpcAccountInfoConfig {
encoding: Some(UiAccountEncoding::Base64),
commitment: Some(CommitmentConfig::processed()),
data_slice: None,
};
// TODO: Make addresses filters configurable
let program_accounts_config = RpcProgramAccountsConfig {
filters: None, /*Some(vec![RpcFilterType::DataSize(
size_of::<MangoAccount>() as u64
)]),*/
with_context: Some(true),
account_config: account_info_config.clone(),
};
let mut update_sub = client
.program_subscribe(
program_id.to_string(),
Some(program_accounts_config.clone()),
)
.map_err_anyhow()?;
let mut slot_sub = client.slots_updates_subscribe().map_err_anyhow()?;
let mut last_snapshot = Instant::now() - snapshot_duration;
loop {
// occasionally cause a new snapshot to be produced
// including the first time
if last_snapshot + snapshot_duration <= Instant::now() {
let account_snapshot = rpc_client
.get_program_accounts(
program_id.to_string(),
Some(program_accounts_config.clone()),
)
.await
.map_err_anyhow()?;
if let OptionalContext::Context(account_snapshot_response) = account_snapshot {
sender
.send(WebsocketMessage::SnapshotUpdate(account_snapshot_response))
.expect("sending must succeed");
}
last_snapshot = Instant::now();
}
tokio::select! {
account = update_sub.next() => {
match account {
Some(account) => {
sender.send(WebsocketMessage::SingleUpdate(account.map_err_anyhow()?)).expect("sending must succeed");
},
None => {
warn!("account stream closed");
return Ok(());
},
}
},
slot_update = slot_sub.next() => {
match slot_update {
Some(slot_update) => {
sender.send(WebsocketMessage::SlotUpdate(slot_update.map_err_anyhow()?)).expect("sending must succeed");
},
None => {
warn!("slot update stream closed");
return Ok(());
},
}
},
_ = tokio::time::sleep(Duration::from_secs(60)) => {
warn!("websocket timeout");
return Ok(())
}
}
}
}
async fn feed_data_accountsdb(
sender: mpsc::Sender<accountsdb_proto::Update>,
) -> Result<(), anyhow::Error> {
let rpc_http_url = "";
let mut client = AccountsDbClient::connect("http://[::1]:10000").await?;
let mut update_stream = client
.subscribe(accountsdb_proto::SubscribeRequest {})
.await?
.into_inner();
let rpc_client = http::connect_with_options::<FullClient>(&rpc_http_url, true)
.await
.map_err_anyhow()?;
let program_id = Pubkey::from_str("mv3ekLzLbnVPNxjSKvqBpU3ZeZXPQdEC3bp5MDEBG68")?;
let account_info_config = RpcAccountInfoConfig {
encoding: Some(UiAccountEncoding::Base64),
commitment: Some(CommitmentConfig::processed()),
data_slice: None,
};
// TODO: Make addresses filters configurable
let program_accounts_config = RpcProgramAccountsConfig {
filters: None, /*Some(vec![RpcFilterType::DataSize(
size_of::<MangoAccount>() as u64
)]),*/
with_context: Some(true),
account_config: account_info_config.clone(),
};
// Get an account snapshot on start
let account_snapshot = rpc_client
.get_program_accounts(
program_id.to_string(),
Some(program_accounts_config.clone()),
)
.await
.map_err_anyhow()?;
if let OptionalContext::Context(account_snapshot_response) = account_snapshot {
// TODO: send the snapshot data through the sender
error!("Missing initial snapshot");
}
loop {
tokio::select! {
update = update_stream.next() => {
match update {
Some(update) => {
sender.send(update?).expect("sending must succeed");
},
None => {
warn!("accountdb stream closed");
return Ok(());
},
}
},
_ = tokio::time::sleep(Duration::from_secs(60)) => {
warn!("accountdb timeout");
return Ok(())
}
}
}
}
#[derive(Clone, PartialEq, Debug)]
pub struct AccountWrite {
pub pubkey: Pubkey,
pub slot: i64,
pub write_version: i64,
pub lamports: i64,
pub owner: Pubkey,
pub executable: bool,
pub rent_epoch: i64,
pub data: Vec<u8>,
}
impl AccountWrite {
fn from(pubkey: Pubkey, slot: u64, write_version: i64, account: Account) -> AccountWrite {
AccountWrite {
pubkey,
slot: slot as i64, // TODO: narrowing!
write_version,
lamports: account.lamports as i64, // TODO: narrowing!
owner: account.owner,
executable: account.executable,
rent_epoch: account.rent_epoch as i64, // TODO: narrowing!
data: account.data.clone(),
}
}
}
#[derive(Clone, PartialEq, Debug)]
struct SlotUpdate {
slot: i64,
parent: Option<i64>,
status: String,
}
fn init_postgres(
connection_string: String,
) -> (mpsc::Sender<AccountWrite>, mpsc::Sender<SlotUpdate>) {
// The actual message may want to also contain a retry count, if it self-reinserts on failure?
let (account_write_queue_sender, account_write_queue_receiver) =
mpsc::channel::<AccountWrite>();
// slot updates are not parallel because their order matters
let (slot_queue_sender, slot_queue_receiver) = mpsc::channel::<SlotUpdate>();
// the postgres connection management thread
// - creates a connection and runs it
// - if it fails, reestablishes it and requests a new snapshot
let postgres_client: Arc<RwLock<Arc<Option<postgres_query::Caching<tokio_postgres::Client>>>>> =
Arc::new(RwLock::new(Arc::new(None)));
let postgres_client_c = Arc::clone(&postgres_client);
let connection_string_c = connection_string.clone();
tokio::spawn(async move {
let (client, connection) =
tokio_postgres::connect(&connection_string_c, tokio_postgres::NoTls)
.await
.unwrap();
{
let mut w = postgres_client_c.write().unwrap();
*w = Arc::new(Some(postgres_query::Caching::new(client)));
}
connection.await.unwrap();
// TODO: on error: log, reconnect, send message that a new snapshot is necessary
});
// separate postgres client for slot updates, because they need transactions
let postgres_client_slots: Arc<
RwLock<Option<postgres_query::Caching<tokio_postgres::Client>>>,
> = Arc::new(RwLock::new(None));
let postgres_client_slots_c = Arc::clone(&postgres_client_slots);
let connection_string_c = connection_string.clone();
tokio::spawn(async move {
let (client, connection) =
tokio_postgres::connect(&connection_string_c, tokio_postgres::NoTls)
.await
.unwrap();
{
let mut w = postgres_client_slots_c.write().unwrap();
*w = Some(postgres_query::Caching::new(client));
}
connection.await.unwrap();
// TODO: on error: log, reconnect, send message that a new snapshot is necessary
});
#[derive(FromSqlRow)]
struct SingleResult(i64);
// postgres account write sending worker thread
let postgres_client_c = Arc::clone(&postgres_client);
tokio::spawn(async move {
loop {
// all of this needs to be in a function, to allow ?
let write = account_write_queue_receiver.recv().unwrap();
// copy the client arc
let client_opt = Arc::clone(&*postgres_client_c.read().unwrap());
// Not sure this is right. What I want is a single thread that calls
// query.fetch(client) and then to process the results of that
// in a bunch of worker threads.
// However, the future returned from query.fetch(client) still requires
// client, so that isn't possible.
// TODO: Nevertheless, there should just be a limited number of these processing threads. Maybe have an intermediary that the worker threads
// send their sender to, then receive work through it?
tokio::spawn(async move {
let client = client_opt.deref().as_ref().unwrap();
let pubkey: &[u8] = &write.pubkey.to_bytes();
let owner: &[u8] = &write.owner.to_bytes();
let query = query!(" \
INSERT INTO account_write \
(pubkey, slot, write_version, owner, lamports, executable, rent_epoch, data) \
VALUES \
($pubkey, $slot, $write_version, $owner, $lamports, $executable, $rent_epoch, $data) \
ON CONFLICT (pubkey, slot, write_version) DO NOTHING", // TODO: should update for same write_version to work with websocket input
pubkey,
slot = write.slot,
write_version = write.write_version,
owner,
lamports = write.lamports,
executable = write.executable,
rent_epoch = write.rent_epoch,
data = write.data,
);
let result = query.execute(client).await.unwrap();
println!("new write: count: {}", result);
});
}
});
// slot update handling thread
let postgres_client_slots_c = Arc::clone(&postgres_client_slots);
tokio::spawn(async move {
let mut slots = HashMap::<i64, SlotUpdate>::new();
let mut newest_nonfinal_slot: Option<i64> = None;
let mut newest_final_slot: Option<i64> = None;
let mut client: Option<postgres_query::Caching<tokio_postgres::Client>> = None;
loop {
let update = slot_queue_receiver.recv().unwrap();
// since we need to mutate the client, move it out of the rwlock here
// TODO: might be easier to understand if we sent the new client over a channel instead
{
let mut lock = postgres_client_slots_c.write().unwrap();
if (*lock).is_some() {
client = (*lock).take();
}
}
let client = client.as_mut().unwrap();
if let Some(parent) = update.parent {
let query = query!(
" \
INSERT INTO slot \
(slot, parent, status, uncle) \
VALUES \
($slot, $parent, $status, FALSE) \
ON CONFLICT (slot) DO UPDATE SET \
parent=$parent, status=$status",
slot = update.slot,
parent = update.parent,
status = update.status,
);
let result = query.execute(client).await.unwrap();
println!("new slot: count: {}", result);
} else {
let query = query!(
" \
INSERT INTO slot \
(slot, parent, status, uncle) \
VALUES \
($slot, NULL, $status, FALSE) \
ON CONFLICT (slot) DO UPDATE SET \
status=$status",
slot = update.slot,
status = update.status,
);
let result = query.execute(client).await.unwrap();
println!("new slot: count: {}", result);
}
if update.status == "rooted" {
println!("slot changed to rooted");
slots.remove(&update.slot);
// TODO: should also convert all parents to rooted, just in case we missed an update
// Keep only the most recent final write per pubkey
if newest_final_slot.unwrap_or(-1) < update.slot {
let query = query!(" \
DELETE FROM account_write \
USING ( \
SELECT DISTINCT ON(pubkey) pubkey, slot, write_version FROM account_write \
WHERE slot <= $newest_final_slot AND status = 'rooted' \
ORDER BY pubkey, slot DESC, write_version DESC \
) latest_write \
WHERE account_write.pubkey = latest_write.pubkey \
AND (account_write.slot < latest_write.slot \
OR (account_write.slot = latest_write.slot \
AND account_write.write_version < latest_write.write_version \
) \
)",
newest_final_slot = update.slot,
);
let result = query.execute(client).await.unwrap();
newest_final_slot = Some(update.slot);
}
if newest_nonfinal_slot.unwrap_or(-1) == update.slot {
newest_nonfinal_slot = None;
}
} else {
let mut parent_update = false;
if let Some(previous) = slots.get_mut(&update.slot) {
previous.status = update.status;
if update.parent.is_some() {
previous.parent = update.parent;
parent_update = true;
}
} else {
slots.insert(update.slot, update.clone());
}
let new_newest_slot = newest_nonfinal_slot.unwrap_or(-1) < update.slot;
if new_newest_slot || parent_update {
println!("recomputing uncles");
// recompute uncles and store to postgres for each slot in slots
let mut uncles: HashMap<i64, bool> = slots.keys().map(|k| (*k, true)).collect();
/* Could do this in SQL like...
with recursive liveslots as (
select * from slot where slot = (select max(slot) from slot)
union
select s.* from slot s inner join liveslots l on l.parent = s.slot where s.status != 'rooted'
)
select * from liveslots order by slot;
*/
let mut it_slot = max(newest_nonfinal_slot.unwrap_or(-1), update.slot);
while let Some(slot) = slots.get(&it_slot) {
let w = uncles
.get_mut(&it_slot)
.expect("uncles has same keys as slots");
println!("{} {:?}", slot.slot, slot.parent);
assert!(*w); // TODO: error instead
*w = false;
if slot.status == "rooted" {
break;
}
if let Some(parent) = slot.parent {
it_slot = parent;
} else {
break;
}
}
// Updating uncle state must be done as a transaction
let transaction = client.transaction().await.unwrap();
for (slot, is_uncle) in uncles {
let query = query!(
"UPDATE slot SET uncle = $is_uncle WHERE slot = $slot",
is_uncle,
slot,
);
let result = query.execute(&transaction).await.unwrap();
}
transaction.into_inner().commit().await.unwrap();
}
if new_newest_slot {
newest_nonfinal_slot = Some(update.slot);
}
}
}
});
(account_write_queue_sender, slot_queue_sender)
}
#[tokio::main]
async fn main() {
log::set_logger(&LOGGER)
.map(|()| log::set_max_level(log::LevelFilter::Info))
.unwrap();
let postgres_connection_string = "host=/var/run/postgresql user=kamm port=5433";
let (account_write_queue_sender, slot_queue_sender) =
init_postgres(postgres_connection_string.into());
/*
sleep(Duration::from_secs(1));
slot_queue_sender.send(SlotUpdate { slot: 1000, parent: 999, status: "processed".into() }).unwrap();
slot_queue_sender.send(SlotUpdate { slot: 1001, parent: 1000, status: "processed".into() }).unwrap();
slot_queue_sender.send(SlotUpdate { slot: 1002, parent: 1001, status: "processed".into() }).unwrap();
slot_queue_sender.send(SlotUpdate { slot: 1003, parent: 1001, status: "processed".into() }).unwrap();
slot_queue_sender.send(SlotUpdate { slot: 1004, parent: 1002, status: "processed".into() }).unwrap();
slot_queue_sender.send(SlotUpdate { slot: 1000, parent: 999, status: "rooted".into() }).unwrap();
*/
// Subscribe to accountsdb
let (update_sender, update_receiver) = mpsc::channel::<accountsdb_proto::Update>();
tokio::spawn(async move {
// Continuously reconnect on failure
loop {
let out = feed_data_accountsdb(update_sender.clone());
let _ = out.await;
}
});
loop {
let update = update_receiver.recv().unwrap();
println!("got update message");
match update.update_oneof.unwrap() {
accountsdb_proto::update::UpdateOneof::AccountWrite(update) => {
println!("single update");
assert!(update.pubkey.len() == 32);
assert!(update.owner.len() == 32);
account_write_queue_sender
.send(AccountWrite {
pubkey: Pubkey::new(&update.pubkey),
slot: update.slot as i64, // TODO: narrowing
write_version: update.write_version as i64,
lamports: update.lamports as i64,
owner: Pubkey::new(&update.owner),
executable: update.executable,
rent_epoch: update.rent_epoch as i64,
data: update.data,
})
.unwrap();
}
accountsdb_proto::update::UpdateOneof::SlotUpdate(update) => {
println!("slot update");
slot_queue_sender
.send(SlotUpdate {
slot: update.slot as i64, // TODO: narrowing
parent: update.parent.map(|v| v as i64),
status: "bla".into(),
})
.unwrap();
}
}
}
return;
// Subscribe to program account updates websocket
let (update_sender, update_receiver) = mpsc::channel::<WebsocketMessage>();
tokio::spawn(async move {
// if the websocket disconnects, we get no data in a while etc, reconnect and try again
loop {
let out = feed_data(update_sender.clone());
let _ = out.await;
}
});
//
// The thread that pulls updates and forwards them to postgres
//
// copy websocket updates into the postgres account write queue
loop {
let update = update_receiver.recv().unwrap();
println!("got update message");
match update {
WebsocketMessage::SingleUpdate(update) => {
println!("single update");
let account: Account = update.value.account.decode().unwrap();
let pubkey = Pubkey::from_str(&update.value.pubkey).unwrap();
account_write_queue_sender
.send(AccountWrite::from(pubkey, update.context.slot, 0, account))
.unwrap();
}
WebsocketMessage::SnapshotUpdate(update) => {
println!("snapshot update");
for keyed_account in update.value {
let account = keyed_account.account.decode().unwrap();
let pubkey = Pubkey::from_str(&keyed_account.pubkey).unwrap();
account_write_queue_sender
.send(AccountWrite::from(pubkey, update.context.slot, 0, account))
.unwrap();
}
}
WebsocketMessage::SlotUpdate(update) => {
println!("slot update");
let message = match *update {
solana_client::rpc_response::SlotUpdate::CreatedBank {
slot, parent, ..
} => Some(SlotUpdate {
slot: slot as i64, // TODO: narrowing
parent: Some(parent as i64),
status: "processed".into(),
}),
solana_client::rpc_response::SlotUpdate::OptimisticConfirmation {
slot,
..
} => Some(SlotUpdate {
slot: slot as i64, // TODO: narrowing
parent: None,
status: "confirmed".into(),
}),
solana_client::rpc_response::SlotUpdate::Root { slot, .. } => {
Some(SlotUpdate {
slot: slot as i64, // TODO: narrowing
parent: None,
status: "rooted".into(),
})
}
_ => None,
};
if let Some(message) = message {
slot_queue_sender.send(message).unwrap();
}
}
}
}
}

45
proto/accountsdb.proto Normal file
View File

@ -0,0 +1,45 @@
syntax = "proto3";
option java_multiple_files = true;
option java_package = "mango.v3.accountsdb";
option java_outer_classname = "AccountsDbProto";
package accountsdb;
service AccountsDb {
rpc Subscribe(SubscribeRequest) returns (stream Update) {}
}
message SubscribeRequest {
}
message Update {
oneof update_oneof {
AccountWrite account_write = 1;
SlotUpdate slot_update = 2;
}
}
message AccountWrite {
uint64 slot = 1;
bytes pubkey = 2;
uint64 lamports = 3;
bytes owner = 4;
bool executable = 5;
uint64 rent_epoch = 6;
bytes data = 7;
uint64 write_version = 8;
bool is_startup = 9;
}
message SlotUpdate {
uint64 slot = 1;
optional uint64 parent = 2;
enum Status {
PROCESSED = 0;
ROOTED = 1;
CONFIRMED = 2;
}
Status status = 3;
}

25
scripts/create_schema.sql Normal file
View File

@ -0,0 +1,25 @@
/**
* This plugin implementation for PostgreSQL requires the following tables
*/
-- The table storing account writes, keeping only the newest write_version per slot
CREATE TABLE account_write (
pubkey BYTEA NOT NULL,
slot BIGINT NOT NULL,
write_version BIGINT NOT NULL,
owner BYTEA,
lamports BIGINT NOT NULL,
executable BOOL NOT NULL,
rent_epoch BIGINT NOT NULL,
data BYTEA,
PRIMARY KEY (pubkey, slot, write_version)
);
-- The table storing slot information
CREATE TABLE slot (
slot BIGINT PRIMARY KEY,
parent BIGINT,
status varchar(16) NOT NULL,
uncle BOOL NOT NULL
);

6
scripts/drop_schema.sql Normal file
View File

@ -0,0 +1,6 @@
/**
* Script for cleaning up the schema for PostgreSQL used for the AccountsDb plugin.
*/
DROP TABLE account_write;
DROP TABLE slot;