web3-proxy/web3_proxy/src/app.rs

1079 lines
43 KiB
Rust
Raw Normal View History

2022-08-10 05:37:34 +03:00
// TODO: this file is way too big now. move things into other modules
2022-08-24 03:59:05 +03:00
use crate::block_number::block_needed;
use crate::config::{AppConfig, TopConfig};
2022-10-10 07:15:07 +03:00
use crate::frontend::authorization::{AuthorizedRequest, RequestMetadata};
2022-08-24 03:59:05 +03:00
use crate::jsonrpc::JsonRpcForwardedResponse;
use crate::jsonrpc::JsonRpcForwardedResponseEnum;
use crate::jsonrpc::JsonRpcRequest;
use crate::jsonrpc::JsonRpcRequestEnum;
use crate::rpcs::blockchain::{ArcBlock, BlockId};
2022-08-30 23:01:42 +03:00
use crate::rpcs::connections::Web3Connections;
2022-09-09 06:53:16 +03:00
use crate::rpcs::request::OpenRequestHandleMetrics;
2022-08-24 03:59:05 +03:00
use crate::rpcs::transactions::TxStatus;
2022-10-10 07:15:07 +03:00
use crate::stats::{ProxyResponseStat, StatEmitter, Web3ProxyStat};
2022-07-26 07:53:38 +03:00
use anyhow::Context;
use axum::extract::ws::Message;
use axum::headers::{Referer, UserAgent};
2022-09-15 20:57:24 +03:00
use deferred_rate_limiter::DeferredRateLimiter;
2022-08-10 05:37:34 +03:00
use derive_more::From;
2022-07-22 22:30:39 +03:00
use ethers::core::utils::keccak256;
2022-08-24 03:59:05 +03:00
use ethers::prelude::{Address, Block, Bytes, TxHash, H256, U64};
use futures::future::Abortable;
2022-05-30 04:28:22 +03:00
use futures::future::{join_all, AbortHandle};
2022-06-14 08:43:28 +03:00
use futures::stream::FuturesUnordered;
2022-05-30 04:28:22 +03:00
use futures::stream::StreamExt;
2022-06-16 20:51:49 +03:00
use futures::Future;
2022-09-09 00:01:36 +03:00
use hashbrown::HashMap;
use ipnet::IpNet;
2022-09-14 10:08:48 +03:00
use metered::{metered, ErrorCount, HitCount, ResponseTime, Throughput};
2022-08-03 03:27:26 +03:00
use migration::{Migrator, MigratorTrait};
2022-09-05 08:53:58 +03:00
use moka::future::Cache;
2022-09-15 20:57:24 +03:00
use redis_rate_limiter::{DeadpoolRuntime, RedisConfig, RedisPool, RedisRateLimiter};
use sea_orm::prelude::Decimal;
use sea_orm::DatabaseConnection;
2022-09-09 06:53:16 +03:00
use serde::Serialize;
2022-05-30 21:23:55 +03:00
use serde_json::json;
2022-05-12 02:50:52 +03:00
use std::fmt;
2022-09-15 20:57:24 +03:00
use std::net::IpAddr;
2022-06-16 20:51:49 +03:00
use std::pin::Pin;
2022-07-14 02:25:01 +03:00
use std::str::FromStr;
2022-09-09 06:53:16 +03:00
use std::sync::atomic::{self, AtomicUsize};
2022-05-12 02:50:52 +03:00
use std::sync::Arc;
use std::time::Duration;
2022-09-27 05:01:45 +03:00
use tokio::sync::{broadcast, watch, Semaphore};
2022-06-14 08:43:28 +03:00
use tokio::task::JoinHandle;
use tokio::time::timeout;
use tokio_stream::wrappers::{BroadcastStream, WatchStream};
use tracing::{error, info, trace, warn};
2022-09-24 08:53:45 +03:00
use ulid::Ulid;
2022-05-12 02:50:52 +03:00
// TODO: make this customizable?
2022-05-12 02:50:52 +03:00
static APP_USER_AGENT: &str = concat!(
"satoshiandkin/",
env!("CARGO_PKG_NAME"),
"/",
env!("CARGO_PKG_VERSION"),
);
2022-08-24 03:59:05 +03:00
/// block hash, method, params
// TODO: better name
2022-09-07 07:11:47 +03:00
type ResponseCacheKey = (H256, String, Option<String>);
type ResponseCache =
Cache<ResponseCacheKey, JsonRpcForwardedResponse, hashbrown::hash_map::DefaultHashBuilder>;
2022-05-21 01:16:15 +03:00
2022-06-14 08:43:28 +03:00
pub type AnyhowJoinHandle<T> = JoinHandle<anyhow::Result<T>>;
#[derive(Clone, Debug, Default, From)]
/// TODO: rename this?
pub struct UserKeyData {
pub user_key_id: u64,
2022-09-28 06:35:55 +03:00
/// if None, allow unlimited queries
pub max_requests_per_period: Option<u64>,
// if None, allow unlimited concurrent requests
pub max_concurrent_requests: Option<u64>,
/// if None, allow any Origin
pub allowed_origins: Option<Vec<String>>,
/// if None, allow any Referer
pub allowed_referers: Option<Vec<Referer>>,
/// if None, allow any UserAgent
pub allowed_user_agents: Option<Vec<UserAgent>>,
/// if None, allow any IP Address
pub allowed_ips: Option<Vec<IpNet>>,
/// Chance to save reverting eth_call, eth_estimateGas, and eth_sendRawTransaction to the database.
pub log_revert_chance: Decimal,
2022-08-10 08:56:09 +03:00
}
2022-08-24 03:59:05 +03:00
/// The application
// TODO: this debug impl is way too verbose. make something smaller
// TODO: i'm sure this is more arcs than necessary, but spawning futures makes references hard
pub struct Web3ProxyApp {
/// Send requests to the best server available
pub balanced_rpcs: Arc<Web3Connections>,
/// Send private requests (like eth_sendRawTransaction) to all these servers
pub private_rpcs: Option<Arc<Web3Connections>>,
2022-09-05 04:52:59 +03:00
response_cache: ResponseCache,
2022-08-24 03:59:05 +03:00
// don't drop this or the sender will stop working
// TODO: broadcast channel instead?
2022-08-30 23:01:42 +03:00
head_block_receiver: watch::Receiver<ArcBlock>,
2022-08-24 03:59:05 +03:00
pending_tx_sender: broadcast::Sender<TxStatus>,
pub config: AppConfig,
pub db_conn: Option<sea_orm::DatabaseConnection>,
2022-09-09 00:01:36 +03:00
/// prometheus metrics
2022-09-09 06:53:16 +03:00
app_metrics: Arc<Web3ProxyAppMetrics>,
open_request_handle_metrics: Arc<OpenRequestHandleMetrics>,
2022-09-05 08:53:58 +03:00
/// store pending transactions that we've seen so that we don't send duplicates to subscribers
pub pending_transactions: Cache<TxHash, TxStatus, hashbrown::hash_map::DefaultHashBuilder>,
2022-09-15 20:57:24 +03:00
pub frontend_ip_rate_limiter: Option<DeferredRateLimiter<IpAddr>>,
2022-09-24 08:53:45 +03:00
pub frontend_key_rate_limiter: Option<DeferredRateLimiter<Ulid>>,
2022-09-24 06:59:21 +03:00
pub login_rate_limiter: Option<RedisRateLimiter>,
pub vredis_pool: Option<RedisPool>,
2022-09-27 05:01:45 +03:00
pub user_key_cache: Cache<Ulid, UserKeyData, hashbrown::hash_map::DefaultHashBuilder>,
pub user_key_semaphores: Cache<u64, Arc<Semaphore>, hashbrown::hash_map::DefaultHashBuilder>,
pub ip_semaphores: Cache<IpAddr, Arc<Semaphore>, hashbrown::hash_map::DefaultHashBuilder>,
pub stat_sender: Option<flume::Sender<Web3ProxyStat>>,
2022-08-24 03:59:05 +03:00
}
2022-07-16 03:08:22 +03:00
/// flatten a JoinError into an anyhow error
2022-08-10 05:37:34 +03:00
/// Useful when joining multiple futures.
2022-06-14 08:43:28 +03:00
pub async fn flatten_handle<T>(handle: AnyhowJoinHandle<T>) -> anyhow::Result<T> {
match handle.await {
Ok(Ok(result)) => Ok(result),
Ok(Err(err)) => Err(err),
Err(err) => Err(err.into()),
}
}
2022-07-16 03:08:22 +03:00
/// return the first error or okay if everything worked
pub async fn flatten_handles<T>(
mut handles: FuturesUnordered<AnyhowJoinHandle<T>>,
2022-06-16 20:51:49 +03:00
) -> anyhow::Result<()> {
while let Some(x) = handles.next().await {
match x {
Err(e) => return Err(e.into()),
Ok(Err(e)) => return Err(e),
2022-07-16 03:08:22 +03:00
Ok(Ok(_)) => continue,
2022-06-16 20:51:49 +03:00
}
}
Ok(())
}
2022-08-10 05:37:34 +03:00
/// Connect to the database and run migrations
pub async fn get_migrated_db(
db_url: String,
2022-09-02 23:16:20 +03:00
min_connections: u32,
2022-08-10 07:30:54 +03:00
max_connections: u32,
) -> anyhow::Result<DatabaseConnection> {
2022-09-14 04:43:09 +03:00
// TODO: scrub credentials and then include the db_url in logs
info!("Connecting to db");
let mut db_opt = sea_orm::ConnectOptions::new(db_url);
2022-08-10 05:37:34 +03:00
// TODO: load all these options from the config file. i think mysql default max is 100
// TODO: sqlx logging only in debug. way too verbose for production
db_opt
2022-09-20 01:41:53 +03:00
.connect_timeout(Duration::from_secs(30))
2022-09-02 23:16:20 +03:00
.min_connections(min_connections)
2022-08-10 07:30:54 +03:00
.max_connections(max_connections)
.sqlx_logging(false);
// .sqlx_logging_level(log::LevelFilter::Info);
let db = sea_orm::Database::connect(db_opt).await?;
// TODO: if error, roll back?
Migrator::up(&db, None).await?;
Ok(db)
}
2022-09-09 06:53:16 +03:00
#[metered(registry = Web3ProxyAppMetrics, registry_expr = self.app_metrics, visibility = pub)]
2022-05-12 02:50:52 +03:00
impl Web3ProxyApp {
2022-10-03 21:08:01 +03:00
/// The main entrypoint.
pub async fn spawn(
2022-08-12 22:07:14 +03:00
top_config: TopConfig,
2022-09-14 09:18:13 +03:00
num_workers: usize,
2022-06-16 20:51:49 +03:00
) -> anyhow::Result<(
Arc<Web3ProxyApp>,
Pin<Box<dyn Future<Output = anyhow::Result<()>>>>,
)> {
2022-08-12 22:07:14 +03:00
// safety checks on the config
assert!(
2022-09-05 09:29:27 +03:00
top_config.app.redirect_user_url.contains("{{user_id}}"),
"redirect user url must contain \"{{user_id}}\""
2022-08-12 22:07:14 +03:00
);
2022-10-03 21:08:01 +03:00
// setup metrics
2022-09-09 06:53:16 +03:00
let app_metrics = Default::default();
let open_request_handle_metrics: Arc<OpenRequestHandleMetrics> = Default::default();
2022-10-03 21:08:01 +03:00
// connect to mysql and make sure the latest migrations have run
let db_conn = if let Some(db_url) = top_config.app.db_url.clone() {
2022-09-14 09:18:13 +03:00
let db_min_connections = top_config
.app
.db_min_connections
.unwrap_or(num_workers as u32);
2022-07-26 07:53:38 +03:00
2022-09-02 23:16:20 +03:00
// TODO: what default multiple?
2022-10-03 21:08:01 +03:00
let db_max_connections = top_config
2022-09-02 23:16:20 +03:00
.app
.db_max_connections
2022-09-06 23:50:37 +03:00
.unwrap_or(db_min_connections * 2);
2022-09-02 23:16:20 +03:00
2022-10-03 21:08:01 +03:00
let db = get_migrated_db(db_url, db_min_connections, db_max_connections).await?;
2022-08-03 03:27:26 +03:00
Some(db)
2022-07-26 07:53:38 +03:00
} else {
info!("no database");
None
};
2022-08-12 22:07:14 +03:00
let balanced_rpcs = top_config.balanced_rpcs;
2022-10-03 21:08:01 +03:00
let private_rpcs = top_config.private_rpcs.unwrap_or_default();
2022-07-09 02:02:32 +03:00
// TODO: try_join_all instead?
2022-06-16 20:51:49 +03:00
let handles = FuturesUnordered::new();
2022-06-14 08:43:28 +03:00
2022-05-12 02:50:52 +03:00
// make a http shared client
2022-07-09 02:02:32 +03:00
// TODO: can we configure the connection pool? should we?
2022-10-03 21:08:01 +03:00
// TODO: timeouts from config. defaults are hopefully good
2022-05-22 02:34:05 +03:00
let http_client = Some(
reqwest::ClientBuilder::new()
.connect_timeout(Duration::from_secs(5))
.timeout(Duration::from_secs(60))
.user_agent(APP_USER_AGENT)
.build()?,
);
2022-10-03 21:08:01 +03:00
// create a connection pool for redis
// a failure to connect does NOT block the application from starting
let vredis_pool = match top_config.app.volatile_redis_url.as_ref() {
2022-07-26 07:53:38 +03:00
Some(redis_url) => {
2022-09-14 04:43:09 +03:00
// TODO: scrub credentials and then include the redis_url in logs
info!("Connecting to vredis");
2022-05-22 21:39:06 +03:00
2022-09-14 09:18:13 +03:00
// TODO: what is a good default?
2022-09-02 23:16:20 +03:00
let redis_max_connections = top_config
.app
.volatile_redis_max_connections
2022-09-14 09:18:13 +03:00
.unwrap_or(num_workers * 2);
// TODO: what are reasonable timeouts?
2022-09-14 09:18:13 +03:00
let redis_pool = RedisConfig::from_url(redis_url)
.builder()?
.max_size(redis_max_connections)
2022-09-15 20:57:24 +03:00
.runtime(DeadpoolRuntime::Tokio1)
2022-09-14 09:18:13 +03:00
.build()?;
// test the redis pool
2022-09-17 04:19:11 +03:00
if let Err(err) = redis_pool.get().await {
error!(
?err,
"failed to connect to vredis. some features will be disabled"
);
2022-09-17 04:19:11 +03:00
};
2022-09-14 09:18:13 +03:00
Some(redis_pool)
2022-05-22 21:39:06 +03:00
}
None => {
warn!("no redis connection. some features will be disabled");
2022-05-22 21:39:06 +03:00
None
2022-05-22 02:34:05 +03:00
}
};
2022-05-12 02:50:52 +03:00
// setup a channel for receiving stats (generally with a high cardinality, such as per-user)
2022-10-03 21:08:01 +03:00
// we do this in a channel so we don't slow down our response to the users
2022-10-10 07:15:07 +03:00
let stat_sender = if let Some(db_conn) = db_conn.clone() {
2022-10-03 21:08:01 +03:00
// TODO: sender and receiver here are a little confusing. because the thing that reads the receiver is what actually submits the stats
let (stat_sender, stat_handle, save_handle) = {
// TODO: period from config instead of always being 60 seconds
2022-10-10 07:15:07 +03:00
let emitter = StatEmitter::new(top_config.app.chain_id, db_conn, 60);
emitter.spawn().await?
};
2022-10-03 21:08:01 +03:00
handles.push(stat_handle);
handles.push(save_handle);
2022-10-03 21:08:01 +03:00
Some(stat_sender)
} else {
warn!("cannot store stats without a redis connection");
2022-10-03 21:08:01 +03:00
None
};
// TODO: i don't like doing Block::default here! Change this to "None"?
2022-07-22 08:11:26 +03:00
let (head_block_sender, head_block_receiver) = watch::channel(Arc::new(Block::default()));
2022-07-09 03:00:31 +03:00
// TODO: will one receiver lagging be okay? how big should this be?
2022-07-26 07:53:38 +03:00
let (pending_tx_sender, pending_tx_receiver) = broadcast::channel(256);
// TODO: use this? it could listen for confirmed transactions and then clear pending_transactions, but the head_block_sender is doing that
2022-10-03 21:08:01 +03:00
// TODO: don't drop the pending_tx_receiver. instead, read it to mark transactions as "seen". once seen, we won't re-send them?
// TODO: once a transaction is "Confirmed" we remove it from the map. this should prevent major memory leaks.
// TODO: we should still have some sort of expiration or maximum size limit for the map
2022-07-26 07:53:38 +03:00
drop(pending_tx_receiver);
2022-09-17 05:30:06 +03:00
// TODO: capacity from configs
// all these are the same size, so no need for a weigher
// TODO: ttl on this? or is max_capacity fine?
let pending_transactions = Cache::builder()
.max_capacity(10_000)
.build_with_hasher(hashbrown::hash_map::DefaultHashBuilder::new());
2022-06-16 20:51:49 +03:00
// keep 1GB of blocks in the cache
2022-09-17 05:30:06 +03:00
// TODO: limits from config
// these blocks don't have full transactions, but they do have rather variable amounts of transaction hashes
// TODO: how can we do the weigher better? this is going to be slow!
let block_map = Cache::builder()
.max_capacity(1024 * 1024 * 1024)
.weigher(|_k, v: &Arc<Block<TxHash>>| {
// TODO: is this good enough?
v.transactions.len().try_into().unwrap_or(u32::MAX)
})
.build_with_hasher(hashbrown::hash_map::DefaultHashBuilder::new());
2022-08-26 20:26:17 +03:00
2022-10-03 21:08:01 +03:00
// connect to the load balanced rpcs
2022-06-14 08:43:28 +03:00
let (balanced_rpcs, balanced_handle) = Web3Connections::spawn(
2022-08-12 22:07:14 +03:00
top_config.app.chain_id,
2022-05-22 02:34:05 +03:00
balanced_rpcs,
http_client.clone(),
vredis_pool.clone(),
2022-08-26 20:26:17 +03:00
block_map.clone(),
Some(head_block_sender),
2022-08-27 06:11:58 +03:00
top_config.app.min_sum_soft_limit,
2022-08-27 03:33:45 +03:00
top_config.app.min_synced_rpcs,
2022-06-16 05:53:37 +03:00
Some(pending_tx_sender.clone()),
2022-06-16 20:51:49 +03:00
pending_transactions.clone(),
2022-09-09 06:53:16 +03:00
open_request_handle_metrics.clone(),
2022-05-22 02:34:05 +03:00
)
2022-07-26 07:53:38 +03:00
.await
2022-10-03 21:08:01 +03:00
.context("spawning balanced rpcs")?;
2022-05-18 19:35:06 +03:00
2022-10-03 21:08:01 +03:00
// save the handle to catch any errors
2022-06-14 08:43:28 +03:00
handles.push(balanced_handle);
2022-10-03 21:08:01 +03:00
// connect to the private rpcs
// only some chains have this, so this is optional
2022-05-12 02:50:52 +03:00
let private_rpcs = if private_rpcs.is_empty() {
// TODO: do None instead of clone?
2022-05-12 02:50:52 +03:00
warn!("No private relays configured. Any transactions will be broadcast to the public mempool!");
None
2022-05-12 02:50:52 +03:00
} else {
2022-06-14 08:43:28 +03:00
let (private_rpcs, private_handle) = Web3Connections::spawn(
2022-08-12 22:07:14 +03:00
top_config.app.chain_id,
2022-05-22 02:34:05 +03:00
private_rpcs,
http_client.clone(),
vredis_pool.clone(),
2022-08-26 20:26:17 +03:00
block_map,
2022-08-11 00:29:50 +03:00
// subscribing to new heads here won't work well. if they are fast, they might be ahead of balanced_rpcs
None,
2022-08-27 03:33:45 +03:00
// minimum doesn't really matter on private rpcs
1,
2022-08-27 06:11:58 +03:00
1,
2022-08-11 00:29:50 +03:00
// TODO: subscribe to pending transactions on the private rpcs? they seem to have low rate limits
None,
2022-06-16 20:51:49 +03:00
pending_transactions.clone(),
2022-09-09 06:53:16 +03:00
open_request_handle_metrics.clone(),
2022-05-22 02:34:05 +03:00
)
2022-07-26 07:53:38 +03:00
.await
2022-10-03 21:08:01 +03:00
.context("spawning private_rpcs")?;
2022-06-14 08:43:28 +03:00
2022-10-03 21:08:01 +03:00
// save the handle to catch any errors
2022-06-14 08:43:28 +03:00
handles.push(private_handle);
Some(private_rpcs)
2022-05-12 02:50:52 +03:00
};
2022-10-03 21:08:01 +03:00
// create rate limiters
// these are optional. they require redis
2022-09-15 20:57:24 +03:00
let mut frontend_ip_rate_limiter = None;
let mut frontend_key_rate_limiter = None;
2022-09-24 06:59:21 +03:00
let mut login_rate_limiter = None;
if let Some(redis_pool) = vredis_pool.as_ref() {
2022-09-24 06:59:21 +03:00
let rpc_rrl = RedisRateLimiter::new(
2022-08-06 08:46:33 +03:00
"web3_proxy",
2022-08-06 08:26:43 +03:00
"frontend",
2022-09-24 06:59:21 +03:00
top_config.app.frontend_rate_limit_per_minute,
2022-08-30 23:01:42 +03:00
60.0,
2022-09-15 20:57:24 +03:00
redis_pool.clone(),
);
2022-09-24 06:59:21 +03:00
// these two rate limiters can share the base limiter
2022-09-15 20:57:24 +03:00
// TODO: take cache_size from config
frontend_ip_rate_limiter = Some(DeferredRateLimiter::<IpAddr>::new(
10_000,
"ip",
2022-09-24 06:59:21 +03:00
rpc_rrl.clone(),
None,
));
2022-09-24 08:53:45 +03:00
frontend_key_rate_limiter = Some(DeferredRateLimiter::<Ulid>::new(
2022-09-24 06:59:21 +03:00
10_000, "key", rpc_rrl, None,
));
// don't defer this one because it will have a low request per peiod
login_rate_limiter = Some(RedisRateLimiter::new(
"web3_proxy",
"login",
top_config.app.login_rate_limit_per_minute,
60.0,
redis_pool.clone(),
2022-09-15 20:57:24 +03:00
));
}
2022-07-07 06:22:09 +03:00
// keep 1GB of blocks in the cache
2022-09-17 05:30:06 +03:00
// responses can be very different in sizes, so this definitely needs a weigher
// TODO: max_capacity from config
// TODO: don't allow any response to be bigger than X% of the cache
let response_cache = Cache::builder()
.max_capacity(1024 * 1024 * 1024)
.weigher(|k: &(H256, String, Option<String>), v| {
// TODO: make this weigher past. serializing json is not fast
let mut size = (k.1).len();
if let Some(params) = &k.2 {
size += params.len()
}
if let Ok(v) = serde_json::to_string(v) {
size += v.len();
// the or in unwrap_or is probably never called
size.try_into().unwrap_or(u32::MAX)
} else {
// this seems impossible
u32::MAX
}
})
.build_with_hasher(hashbrown::hash_map::DefaultHashBuilder::new());
2022-09-15 20:57:24 +03:00
2022-09-17 05:30:06 +03:00
// all the users are the same size, so no need for a weigher
2022-10-03 21:08:01 +03:00
// if there is no database of users, there will be no keys and so this will be empty
2022-09-17 05:30:06 +03:00
// TODO: max_capacity from config
2022-09-22 02:50:55 +03:00
// TODO: ttl from config
2022-09-27 05:01:45 +03:00
let user_key_cache = Cache::builder()
.max_capacity(10_000)
2022-09-20 02:09:31 +03:00
.time_to_live(Duration::from_secs(60))
.build_with_hasher(hashbrown::hash_map::DefaultHashBuilder::new());
2022-09-05 08:53:58 +03:00
2022-10-03 21:08:01 +03:00
// create semaphores for concurrent connection limits
2022-09-27 05:01:45 +03:00
// TODO: what should tti be for semaphores?
let user_key_semaphores = Cache::builder()
.time_to_idle(Duration::from_secs(120))
.build_with_hasher(hashbrown::hash_map::DefaultHashBuilder::new());
let ip_semaphores = Cache::builder()
.time_to_idle(Duration::from_secs(120))
.build_with_hasher(hashbrown::hash_map::DefaultHashBuilder::new());
2022-07-07 06:22:09 +03:00
let app = Self {
2022-08-12 22:07:14 +03:00
config: top_config.app,
2022-05-13 23:50:11 +03:00
balanced_rpcs,
2022-05-12 02:50:52 +03:00
private_rpcs,
2022-09-05 08:53:58 +03:00
response_cache,
2022-05-30 04:28:22 +03:00
head_block_receiver,
2022-06-16 05:53:37 +03:00
pending_tx_sender,
2022-06-16 20:51:49 +03:00
pending_transactions,
2022-09-15 20:57:24 +03:00
frontend_ip_rate_limiter,
frontend_key_rate_limiter,
2022-09-24 06:59:21 +03:00
login_rate_limiter,
2022-07-26 07:53:38 +03:00
db_conn,
vredis_pool,
2022-09-09 06:53:16 +03:00
app_metrics,
open_request_handle_metrics,
2022-09-27 05:01:45 +03:00
user_key_cache,
user_key_semaphores,
ip_semaphores,
stat_sender,
};
let app = Arc::new(app);
2022-06-16 20:51:49 +03:00
let handle = Box::pin(flatten_handles(handles));
2022-06-14 08:43:28 +03:00
Ok((app, handle))
2022-05-12 02:50:52 +03:00
}
2022-09-10 05:59:07 +03:00
pub fn prometheus_metrics(&self) -> String {
2022-09-09 00:01:36 +03:00
let globals = HashMap::new();
// TODO: what globals? should this be the hostname or what?
// globals.insert("service", "web3_proxy");
2022-09-09 06:53:16 +03:00
#[derive(Serialize)]
struct CombinedMetrics<'a> {
app: &'a Web3ProxyAppMetrics,
backend_rpc: &'a OpenRequestHandleMetrics,
}
let metrics = CombinedMetrics {
app: &self.app_metrics,
backend_rpc: &self.open_request_handle_metrics,
};
2022-09-10 05:59:07 +03:00
serde_prometheus::to_string(&metrics, Some("web3_proxy"), globals)
.expect("prometheus metrics should always serialize")
2022-09-09 00:01:36 +03:00
}
2022-09-14 10:08:48 +03:00
#[measure([ErrorCount, HitCount, ResponseTime, Throughput])]
2022-09-09 00:01:36 +03:00
pub async fn eth_subscribe<'a>(
self: &'a Arc<Self>,
2022-09-24 08:53:45 +03:00
authorized_request: Arc<AuthorizedRequest>,
payload: JsonRpcRequest,
2022-09-09 00:01:36 +03:00
subscription_count: &'a AtomicUsize,
2022-05-30 04:28:22 +03:00
// TODO: taking a sender for Message instead of the exact json we are planning to send feels wrong, but its easier for now
2022-07-09 01:14:45 +03:00
response_sender: flume::Sender<Message>,
2022-05-30 04:28:22 +03:00
) -> anyhow::Result<(AbortHandle, JsonRpcForwardedResponse)> {
2022-06-14 10:13:42 +03:00
let (subscription_abort_handle, subscription_registration) = AbortHandle::new_pair();
2022-05-30 04:28:22 +03:00
// TODO: this only needs to be unique per connection. we don't need it globably unique
2022-07-09 01:14:45 +03:00
let subscription_id = subscription_count.fetch_add(1, atomic::Ordering::SeqCst);
2022-08-04 02:17:02 +03:00
let subscription_id = U64::from(subscription_id);
2022-05-30 21:23:55 +03:00
2022-06-05 23:39:58 +03:00
// save the id so we can use it in the response
let id = payload.id.clone();
2022-07-16 03:35:54 +03:00
// TODO: calling json! on every request is probably not fast. but we can only match against
// TODO: i think we need a stricter EthSubscribeRequest type that JsonRpcRequest can turn into
match payload.params {
Some(x) if x == json!(["newHeads"]) => {
let head_block_receiver = self.head_block_receiver.clone();
trace!(?subscription_id, "new heads subscription");
tokio::spawn(async move {
let mut head_block_receiver = Abortable::new(
WatchStream::new(head_block_receiver),
subscription_registration,
);
while let Some(new_head) = head_block_receiver.next().await {
// TODO: make a struct for this? using our JsonRpcForwardedResponse won't work because it needs an id
let msg = json!({
"jsonrpc": "2.0",
"method":"eth_subscription",
"params": {
"subscription": subscription_id,
2022-08-11 00:29:50 +03:00
// TODO: option to include full transaction objects instead of just the hashes?
2022-07-22 08:11:26 +03:00
"result": new_head.as_ref(),
},
});
2022-08-11 00:29:50 +03:00
// TODO: do clients support binary messages?
let msg = Message::Text(
serde_json::to_string(&msg).expect("this should always be valid json"),
);
2022-07-09 01:14:45 +03:00
if response_sender.send_async(msg).await.is_err() {
// TODO: cancel this subscription earlier? select on head_block_receiver.next() and an abort handle?
break;
};
}
trace!(?subscription_id, "closed new heads subscription");
});
2022-05-30 04:28:22 +03:00
}
Some(x) if x == json!(["newPendingTransactions"]) => {
let pending_tx_receiver = self.pending_tx_sender.subscribe();
let mut pending_tx_receiver = Abortable::new(
BroadcastStream::new(pending_tx_receiver),
subscription_registration,
);
trace!(?subscription_id, "pending transactions subscription");
tokio::spawn(async move {
while let Some(Ok(new_tx_state)) = pending_tx_receiver.next().await {
let new_tx = match new_tx_state {
2022-08-24 03:59:05 +03:00
TxStatus::Pending(tx) => tx,
TxStatus::Confirmed(..) => continue,
TxStatus::Orphaned(tx) => tx,
};
// TODO: make a struct for this? using our JsonRpcForwardedResponse won't work because it needs an id
let msg = json!({
"jsonrpc": "2.0",
"method": "eth_subscription",
"params": {
"subscription": subscription_id,
"result": new_tx.hash,
},
});
let msg =
Message::Text(serde_json::to_string(&msg).expect("we made this `msg`"));
2022-07-09 01:14:45 +03:00
if response_sender.send_async(msg).await.is_err() {
// TODO: cancel this subscription earlier? select on head_block_receiver.next() and an abort handle?
break;
};
}
trace!(?subscription_id, "closed new heads subscription");
});
}
Some(x) if x == json!(["newPendingFullTransactions"]) => {
// TODO: too much copy/pasta with newPendingTransactions
let pending_tx_receiver = self.pending_tx_sender.subscribe();
let mut pending_tx_receiver = Abortable::new(
BroadcastStream::new(pending_tx_receiver),
subscription_registration,
);
trace!(?subscription_id, "pending transactions subscription");
// TODO: do something with this handle?
tokio::spawn(async move {
while let Some(Ok(new_tx_state)) = pending_tx_receiver.next().await {
let new_tx = match new_tx_state {
2022-08-24 03:59:05 +03:00
TxStatus::Pending(tx) => tx,
TxStatus::Confirmed(..) => continue,
TxStatus::Orphaned(tx) => tx,
};
// TODO: make a struct for this? using our JsonRpcForwardedResponse won't work because it needs an id
let msg = json!({
"jsonrpc": "2.0",
"method": "eth_subscription",
"params": {
"subscription": subscription_id,
// upstream just sends the txid, but we want to send the whole transaction
"result": new_tx,
},
});
let msg = Message::Text(
serde_json::to_string(&msg).expect("we made this message"),
);
2022-07-09 01:14:45 +03:00
if response_sender.send_async(msg).await.is_err() {
2022-06-18 10:06:54 +03:00
// TODO: cancel this subscription earlier? select on head_block_receiver.next() and an abort handle?
break;
};
}
trace!(?subscription_id, "closed new heads subscription");
});
}
Some(x) if x == json!(["newPendingRawTransactions"]) => {
2022-06-18 10:06:54 +03:00
// TODO: too much copy/pasta with newPendingTransactions
let pending_tx_receiver = self.pending_tx_sender.subscribe();
let mut pending_tx_receiver = Abortable::new(
BroadcastStream::new(pending_tx_receiver),
subscription_registration,
);
trace!(?subscription_id, "pending transactions subscription");
// TODO: do something with this handle?
tokio::spawn(async move {
while let Some(Ok(new_tx_state)) = pending_tx_receiver.next().await {
let new_tx = match new_tx_state {
2022-08-24 03:59:05 +03:00
TxStatus::Pending(tx) => tx,
TxStatus::Confirmed(..) => continue,
TxStatus::Orphaned(tx) => tx,
2022-06-18 10:06:54 +03:00
};
// TODO: make a struct for this? using our JsonRpcForwardedResponse won't work because it needs an id
let msg = json!({
"jsonrpc": "2.0",
"method": "eth_subscription",
"params": {
"subscription": subscription_id,
// upstream just sends the txid, but we want to send the raw transaction
2022-06-18 10:06:54 +03:00
"result": new_tx.rlp(),
},
});
2022-09-30 07:18:18 +03:00
let msg = Message::Text(
serde_json::to_string(&msg).expect("this message was just built"),
);
2022-06-18 10:06:54 +03:00
2022-07-09 01:14:45 +03:00
if response_sender.send_async(msg).await.is_err() {
// TODO: cancel this subscription earlier? select on head_block_receiver.next() and an abort handle?
break;
};
}
trace!(?subscription_id, "closed new heads subscription");
});
}
_ => return Err(anyhow::anyhow!("unimplemented")),
}
2022-05-30 04:28:22 +03:00
2022-06-14 10:13:42 +03:00
// TODO: do something with subscription_join_handle?
2022-05-30 04:28:22 +03:00
2022-07-22 22:30:39 +03:00
let response = JsonRpcForwardedResponse::from_value(json!(subscription_id), id);
2022-05-30 04:28:22 +03:00
2022-06-16 05:53:37 +03:00
// TODO: make a `SubscriptonHandle(AbortHandle, JoinHandle)` struct?
2022-06-14 10:13:42 +03:00
Ok((subscription_abort_handle, response))
}
2022-07-22 22:30:39 +03:00
/// send the request or batch of requests to the approriate RPCs
2022-05-12 02:50:52 +03:00
pub async fn proxy_web3_rpc(
2022-09-09 00:01:36 +03:00
self: &Arc<Self>,
2022-10-10 07:15:07 +03:00
authorized_request: Arc<AuthorizedRequest>,
2022-05-12 02:50:52 +03:00
request: JsonRpcRequestEnum,
2022-05-20 08:27:18 +03:00
) -> anyhow::Result<JsonRpcForwardedResponseEnum> {
2022-09-05 04:52:59 +03:00
// TODO: this should probably be trace level
trace!(?request, "proxy_web3_rpc");
2022-05-12 02:50:52 +03:00
// even though we have timeouts on the requests to our backend providers,
2022-07-22 22:30:39 +03:00
// we need a timeout for the incoming request so that retries don't run forever
// TODO: take this as an optional argument. per user max? expiration time instead of duration?
let max_time = Duration::from_secs(120);
2022-05-12 02:50:52 +03:00
let response = match request {
JsonRpcRequestEnum::Single(request) => JsonRpcForwardedResponseEnum::Single(
timeout(
max_time,
2022-09-24 08:53:45 +03:00
self.proxy_web3_rpc_request(authorized_request, request),
)
.await??,
),
JsonRpcRequestEnum::Batch(requests) => JsonRpcForwardedResponseEnum::Batch(
timeout(
max_time,
2022-09-24 08:53:45 +03:00
self.proxy_web3_rpc_requests(authorized_request, requests),
)
.await??,
),
2022-05-12 02:50:52 +03:00
};
2022-09-05 04:52:59 +03:00
// TODO: this should probably be trace level
trace!(?response, "Forwarding");
2022-05-17 03:56:56 +03:00
2022-05-20 08:27:18 +03:00
Ok(response)
2022-05-12 02:50:52 +03:00
}
2022-09-09 00:01:36 +03:00
/// cut up the request and send to potentually different servers
/// TODO: make sure this isn't a problem
2022-05-12 02:50:52 +03:00
async fn proxy_web3_rpc_requests(
2022-09-09 00:01:36 +03:00
self: &Arc<Self>,
2022-10-10 07:15:07 +03:00
authorized_request: Arc<AuthorizedRequest>,
2022-05-12 02:50:52 +03:00
requests: Vec<JsonRpcRequest>,
) -> anyhow::Result<Vec<JsonRpcForwardedResponse>> {
// TODO: we should probably change ethers-rs to support this directly
let num_requests = requests.len();
2022-10-10 07:15:07 +03:00
2022-05-12 02:50:52 +03:00
let responses = join_all(
requests
.into_iter()
2022-10-10 07:15:07 +03:00
.map(|request| {
let authorized_request = authorized_request.clone();
// TODO: spawn so the requests go in parallel
// TODO: i think we will need to flatten
self.proxy_web3_rpc_request(authorized_request, request)
})
2022-05-12 02:50:52 +03:00
.collect::<Vec<_>>(),
)
.await;
// TODO: i'm sure this could be done better with iterators
let mut collected: Vec<JsonRpcForwardedResponse> = Vec::with_capacity(num_requests);
for response in responses {
2022-05-18 19:35:06 +03:00
collected.push(response?);
2022-05-12 02:50:52 +03:00
}
Ok(collected)
}
2022-09-24 07:31:06 +03:00
pub fn db_conn(&self) -> Option<&DatabaseConnection> {
self.db_conn.as_ref()
}
2022-09-15 20:57:24 +03:00
pub async fn redis_conn(&self) -> anyhow::Result<redis_rate_limiter::RedisConnection> {
match self.vredis_pool.as_ref() {
None => Err(anyhow::anyhow!("no redis server configured")),
Some(redis_pool) => {
let redis_conn = redis_pool.get().await?;
Ok(redis_conn)
}
}
}
2022-09-14 10:08:48 +03:00
#[measure([ErrorCount, HitCount, ResponseTime, Throughput])]
2022-05-12 02:50:52 +03:00
async fn proxy_web3_rpc_request(
2022-09-09 00:01:36 +03:00
self: &Arc<Self>,
2022-10-10 07:15:07 +03:00
authorized_request: Arc<AuthorizedRequest>,
mut request: JsonRpcRequest,
2022-05-12 02:50:52 +03:00
) -> anyhow::Result<JsonRpcForwardedResponse> {
trace!("Received request: {:?}", request);
// TODO: allow customizing the period?
let request_metadata = Arc::new(RequestMetadata::new(60, &request)?);
2022-10-10 07:15:07 +03:00
// save the id so we can attach it to the response
// TODO: instead of cloning, take the id out
2022-09-07 06:54:16 +03:00
let request_id = request.id.clone();
2022-05-31 04:55:04 +03:00
// TODO: if eth_chainId or net_version, serve those without querying the backend
2022-09-22 02:50:55 +03:00
// TODO: don't clone?
2022-09-05 08:53:58 +03:00
let partial_response: serde_json::Value = match request.method.clone().as_ref() {
// lots of commands are blocked
2022-09-09 00:01:36 +03:00
method @ ("admin_addPeer"
2022-06-14 09:54:19 +03:00
| "admin_datadir"
| "admin_startRPC"
| "admin_startWS"
| "admin_stopRPC"
| "admin_stopWS"
| "db_getHex"
| "db_getString"
| "db_putHex"
| "db_putString"
2022-06-14 09:54:19 +03:00
| "debug_chaindbCompact"
| "debug_freezeClient"
| "debug_goTrace"
| "debug_mutexProfile"
| "debug_setBlockProfileRate"
| "debug_setGCPercent"
| "debug_setHead"
| "debug_setMutexProfileFraction"
| "debug_standardTraceBlockToFile"
| "debug_standardTraceBadBlockToFile"
| "debug_startCPUProfile"
| "debug_startGoTrace"
| "debug_stopCPUProfile"
| "debug_stopGoTrace"
| "debug_writeBlockProfile"
| "debug_writeMemProfile"
| "debug_writeMutexProfile"
| "eth_compileLLL"
| "eth_compileSerpent"
| "eth_compileSolidity"
| "eth_getCompilers"
| "eth_sendTransaction"
| "eth_sign"
| "eth_signTransaction"
| "eth_submitHashrate"
| "eth_submitWork"
2022-06-14 09:54:19 +03:00
| "les_addBalance"
| "les_setClientParams"
| "les_setDefaultParams"
| "miner_setExtra"
| "miner_setGasPrice"
| "miner_start"
| "miner_stop"
| "miner_setEtherbase"
| "miner_setGasLimit"
| "personal_importRawKey"
| "personal_listAccounts"
| "personal_lockAccount"
| "personal_newAccount"
| "personal_unlockAccount"
| "personal_sendTransaction"
| "personal_sign"
| "personal_ecRecover"
| "shh_addToGroup"
| "shh_getFilterChanges"
| "shh_getMessages"
| "shh_hasIdentity"
| "shh_newFilter"
| "shh_newGroup"
| "shh_newIdentity"
| "shh_post"
| "shh_uninstallFilter"
2022-09-09 00:01:36 +03:00
| "shh_version") => {
// TODO: client error stat
2022-06-30 03:52:04 +03:00
// TODO: proper error code
2022-09-09 00:01:36 +03:00
return Err(anyhow::anyhow!("method unsupported: {}", method));
}
// TODO: implement these commands
2022-09-09 00:01:36 +03:00
method @ ("eth_getFilterChanges"
| "eth_getFilterLogs"
| "eth_newBlockFilter"
| "eth_newFilter"
| "eth_newPendingTransactionFilter"
2022-09-09 00:01:36 +03:00
| "eth_uninstallFilter") => {
// TODO: unsupported command stat
2022-09-09 00:01:36 +03:00
return Err(anyhow::anyhow!("not yet implemented: {}", method));
}
// some commands can use local data or caches
"eth_accounts" => {
// no stats on this. its cheap
serde_json::Value::Array(vec![])
}
"eth_blockNumber" => {
2022-09-01 08:58:55 +03:00
match self.balanced_rpcs.head_block_num() {
Some(head_block_num) => {
json!(head_block_num)
}
None => {
// TODO: what does geth do if this happens?
2022-09-09 00:01:36 +03:00
return Err(anyhow::anyhow!(
"no servers synced. unknown eth_blockNumber"
));
2022-09-01 08:58:55 +03:00
}
}
}
// TODO: eth_callBundle (https://docs.flashbots.net/flashbots-auction/searchers/advanced/rpc-endpoint#eth_callbundle)
// TODO: eth_cancelPrivateTransaction (https://docs.flashbots.net/flashbots-auction/searchers/advanced/rpc-endpoint#eth_cancelprivatetransaction, but maybe just reject)
// TODO: eth_sendPrivateTransaction (https://docs.flashbots.net/flashbots-auction/searchers/advanced/rpc-endpoint#eth_sendprivatetransaction)
"eth_coinbase" => {
2022-07-22 22:30:39 +03:00
// no need for serving coinbase
// we could return a per-user payment address here, but then we might leak that to dapps
// no stats on this. its cheap
2022-07-22 22:30:39 +03:00
json!(Address::zero())
}
// TODO: eth_estimateGas using anvil?
// TODO: eth_gasPrice that does awesome magic to predict the future
"eth_hashrate" => {
// no stats on this. its cheap
2022-07-22 22:30:39 +03:00
json!(U64::zero())
2022-06-30 03:52:04 +03:00
}
"eth_mining" => {
// no stats on this. its cheap
2022-07-22 22:30:39 +03:00
json!(false)
}
// TODO: eth_sendBundle (flashbots command)
// broadcast transactions to all private rpcs at once
2022-08-09 19:54:05 +03:00
"eth_sendRawTransaction" => {
// emit stats
let rpcs = self.private_rpcs.as_ref().unwrap_or(&self.balanced_rpcs);
return rpcs
.try_send_all_upstream_servers(
Some(&authorized_request),
request,
Some(request_metadata),
None,
)
.await;
2022-08-09 19:54:05 +03:00
}
"eth_syncing" => {
// no stats on this. its cheap
// TODO: return a real response if all backends are syncing or if no servers in sync
2022-07-22 22:30:39 +03:00
json!(false)
}
"net_listening" => {
// no stats on this. its cheap
// TODO: only if there are some backends on balanced_rpcs?
2022-07-22 22:30:39 +03:00
json!(true)
}
"net_peerCount" => {
// emit stats
self.balanced_rpcs.num_synced_rpcs().into()
}
"web3_clientVersion" => {
// no stats on this. its cheap
serde_json::Value::String(APP_USER_AGENT.to_string())
}
2022-07-22 22:30:39 +03:00
"web3_sha3" => {
// emit stats
2022-07-22 22:30:39 +03:00
// returns Keccak-256 (not the standardized SHA3-256) of the given data.
match &request.params {
Some(serde_json::Value::Array(params)) => {
2022-08-10 05:37:34 +03:00
// TODO: make a struct and use serde conversion to clean this up
2022-07-22 22:30:39 +03:00
if params.len() != 1 || !params[0].is_string() {
2022-09-10 03:58:33 +03:00
// TODO: this needs the correct error code in the response
2022-07-22 22:30:39 +03:00
return Err(anyhow::anyhow!("invalid request"));
}
let param = Bytes::from_str(
params[0]
.as_str()
.context("parsing params 0 into str then bytes")?,
)?;
2022-07-22 22:30:39 +03:00
let hash = H256::from(keccak256(param));
2022-07-22 22:30:39 +03:00
json!(hash)
}
2022-09-10 03:58:33 +03:00
_ => {
// TODO: this needs the correct error code in the response
return Err(anyhow::anyhow!("invalid request"));
}
2022-07-22 22:30:39 +03:00
}
}
2022-07-22 22:30:39 +03:00
// anything else gets sent to backend rpcs and cached
2022-06-30 03:52:04 +03:00
method => {
// emit stats
2022-09-07 06:54:16 +03:00
// TODO: wait for them to be synced?
let head_block_id = self
2022-09-01 08:58:55 +03:00
.balanced_rpcs
2022-09-07 06:54:16 +03:00
.head_block_id()
2022-09-01 08:58:55 +03:00
.context("no servers synced")?;
2022-07-16 07:13:02 +03:00
// we do this check before checking caches because it might modify the request params
2022-07-16 08:48:02 +03:00
// TODO: add a stat for archive vs full since they should probably cost different
2022-09-22 02:50:55 +03:00
let request_block_id = if let Some(request_block_needed) = block_needed(
method,
request.params.as_mut(),
head_block_id.num,
&self.balanced_rpcs,
)
2022-09-30 07:18:18 +03:00
.await?
2022-09-05 09:13:36 +03:00
{
2022-09-07 06:54:16 +03:00
// TODO: maybe this should be on the app and not on balanced_rpcs
let request_block_hash =
self.balanced_rpcs.block_hash(&request_block_needed).await?;
2022-09-05 08:53:58 +03:00
2022-09-07 06:54:16 +03:00
BlockId {
num: request_block_needed,
hash: request_block_hash,
}
2022-09-07 06:54:16 +03:00
} else {
head_block_id
};
2022-09-07 06:54:16 +03:00
// TODO: struct for this?
// TODO: this can be rather large. is that okay?
let cache_key = (
request_block_id.hash,
request.method.clone(),
request.params.clone().map(|x| x.to_string()),
);
let mut response = {
let request_metadata = request_metadata.clone();
2022-10-10 07:15:07 +03:00
let authorized_request = authorized_request.clone();
self.response_cache
.try_get_with(cache_key, async move {
// TODO: retry some failures automatically!
// TODO: try private_rpcs if all the balanced_rpcs fail!
// TODO: put the hash here instead?
let mut response = self
.balanced_rpcs
.try_send_best_upstream_server(
2022-10-10 07:15:07 +03:00
Some(&authorized_request),
request,
Some(&request_metadata),
Some(&request_block_id.num),
)
.await?;
// discard their id by replacing it with an empty
response.id = Default::default();
Ok::<_, anyhow::Error>(response)
})
.await
// TODO: what is the best way to handle an Arc here?
.map_err(|err| {
// TODO: emit a stat for an error
anyhow::anyhow!(err)
})
.context("caching response")?
};
// since this data came likely out of a cache, the id is not going to match
// replace the id with our request's id.
// TODO: cache without the id
response.id = request_id;
2022-10-10 07:15:07 +03:00
if let (Some(stat_sender), Ok(AuthorizedRequest::User(Some(_), authorized_key))) = (
self.stat_sender.as_ref(),
Arc::try_unwrap(authorized_request),
) {
let response_stat = ProxyResponseStat::new(
method.to_string(),
2022-10-10 07:15:07 +03:00
authorized_key,
request_metadata,
&response,
);
stat_sender.send_async(response_stat.into()).await?;
}
2022-05-29 04:23:58 +03:00
2022-07-22 22:30:39 +03:00
return Ok(response);
}
2022-07-22 22:30:39 +03:00
};
2022-09-07 06:54:16 +03:00
let response = JsonRpcForwardedResponse::from_value(partial_response, request_id);
2022-07-22 22:30:39 +03:00
Ok(response)
2022-05-12 02:50:52 +03:00
}
}
2022-08-24 03:59:05 +03:00
impl fmt::Debug for Web3ProxyApp {
fn fmt(&self, f: &mut fmt::Formatter<'_>) -> fmt::Result {
// TODO: the default formatter takes forever to write. this is too quiet though
f.debug_struct("Web3ProxyApp").finish_non_exhaustive()
}
}