mirror of
https://github.com/Cuprate/cuprate.git
synced 2024-11-16 15:58:17 +00:00
change epee-encoding to monero-epee-bin-serde
This commit is contained in:
parent
20f6af7951
commit
bfbafa4ed5
15 changed files with 403 additions and 703 deletions
|
@ -15,7 +15,7 @@ use monero_consensus::{
|
|||
Database, DatabaseRequest, DatabaseResponse,
|
||||
};
|
||||
|
||||
const BATCH_SIZE: u64 = MAX_BLOCKS_IN_RANGE * 4;
|
||||
const BATCH_SIZE: u64 = MAX_BLOCKS_IN_RANGE * 3;
|
||||
|
||||
/// A cache which can keep chain state while scanning.
|
||||
///
|
||||
|
@ -153,17 +153,7 @@ async fn main() {
|
|||
"http://xmr-node.cakewallet.com:18081".to_string(),
|
||||
"http://node.sethforprivacy.com".to_string(),
|
||||
"http://nodex.monerujo.io:18081".to_string(),
|
||||
//"http://node.community.rino.io:18081".to_string(),
|
||||
"http://nodes.hashvault.pro:18081".to_string(),
|
||||
// "http://node.moneroworld.com:18089".to_string(),
|
||||
"http://node.c3pool.com:18081".to_string(),
|
||||
//
|
||||
"http://xmr-node.cakewallet.com:18081".to_string(),
|
||||
"http://node.sethforprivacy.com".to_string(),
|
||||
"http://nodex.monerujo.io:18081".to_string(),
|
||||
//"http://node.community.rino.io:18081".to_string(),
|
||||
"http://nodes.hashvault.pro:18081".to_string(),
|
||||
// "http://node.moneroworld.com:18089".to_string(),
|
||||
"http://node.c3pool.com:18081".to_string(),
|
||||
];
|
||||
|
||||
|
|
|
@ -13,6 +13,7 @@ use serde_json::json;
|
|||
use tower::balance::p2c::Balance;
|
||||
use tower::util::BoxService;
|
||||
use tower::ServiceExt;
|
||||
use tracing::Instrument;
|
||||
|
||||
use cuprate_common::BlockID;
|
||||
use monero_wire::common::{BlockCompleteEntry, TransactionBlobs};
|
||||
|
@ -22,7 +23,9 @@ use crate::block::weight::BlockWeightInfo;
|
|||
use crate::hardforks::BlockHFInfo;
|
||||
use crate::{DatabaseRequest, DatabaseResponse};
|
||||
|
||||
pub const MAX_BLOCKS_IN_RANGE: u64 = 75;
|
||||
mod discover;
|
||||
|
||||
pub const MAX_BLOCKS_IN_RANGE: u64 = 200;
|
||||
pub const MAX_BLOCKS_HEADERS_IN_RANGE: u64 = 200;
|
||||
|
||||
#[derive(Clone)]
|
||||
|
@ -213,15 +216,17 @@ enum RpcState<R: RpcConnection> {
|
|||
}
|
||||
pub struct Rpc<R: RpcConnection> {
|
||||
rpc: Arc<futures::lock::Mutex<monero_serai::rpc::Rpc<R>>>,
|
||||
addr: String,
|
||||
rpc_state: RpcState<R>,
|
||||
error_slot: Arc<Mutex<Option<RpcError>>>,
|
||||
}
|
||||
|
||||
impl Rpc<HttpRpc> {
|
||||
pub fn new_http(addr: String) -> Rpc<HttpRpc> {
|
||||
let http_rpc = HttpRpc::new(addr).unwrap();
|
||||
let http_rpc = HttpRpc::new(addr.clone()).unwrap();
|
||||
Rpc {
|
||||
rpc: Arc::new(futures::lock::Mutex::new(http_rpc)),
|
||||
addr,
|
||||
rpc_state: RpcState::Locked,
|
||||
error_slot: Arc::new(Mutex::new(None)),
|
||||
}
|
||||
|
@ -257,19 +262,22 @@ impl<R: RpcConnection + Send + Sync + 'static> tower::Service<DatabaseRequest> f
|
|||
panic!("poll_ready was not called first!");
|
||||
};
|
||||
|
||||
let span = tracing::info_span!("rpc_request", addr = &self.addr);
|
||||
|
||||
let err_slot = self.error_slot.clone();
|
||||
|
||||
match req {
|
||||
DatabaseRequest::BlockHash(height) => async move {
|
||||
let res: Result<_, RpcError> = rpc
|
||||
.get_block_hash(height as usize)
|
||||
.map_ok(|hash| DatabaseResponse::BlockHash(hash))
|
||||
.map_ok(DatabaseResponse::BlockHash)
|
||||
.await;
|
||||
if let Err(e) = &res {
|
||||
*err_slot.lock().unwrap() = Some(e.clone());
|
||||
}
|
||||
res.map_err(Into::into)
|
||||
}
|
||||
.instrument(span)
|
||||
.boxed(),
|
||||
DatabaseRequest::ChainHeight => async move {
|
||||
let res: Result<_, RpcError> = rpc
|
||||
|
@ -281,21 +289,32 @@ impl<R: RpcConnection + Send + Sync + 'static> tower::Service<DatabaseRequest> f
|
|||
}
|
||||
res.map_err(Into::into)
|
||||
}
|
||||
.instrument(span)
|
||||
.boxed(),
|
||||
|
||||
DatabaseRequest::BlockPOWInfo(id) => get_blocks_pow_info(id, rpc).boxed(),
|
||||
DatabaseRequest::BlockWeights(id) => get_blocks_weight_info(id, rpc).boxed(),
|
||||
DatabaseRequest::BlockHFInfo(id) => get_blocks_hf_info(id, rpc).boxed(),
|
||||
DatabaseRequest::BlockHfInfoInRange(range) => {
|
||||
get_blocks_hf_info_in_range(range, rpc).boxed()
|
||||
DatabaseRequest::BlockPOWInfo(id) => {
|
||||
get_blocks_pow_info(id, rpc).instrument(span).boxed()
|
||||
}
|
||||
DatabaseRequest::BlockWeights(id) => {
|
||||
get_blocks_weight_info(id, rpc).instrument(span).boxed()
|
||||
}
|
||||
DatabaseRequest::BlockHFInfo(id) => {
|
||||
get_blocks_hf_info(id, rpc).instrument(span).boxed()
|
||||
}
|
||||
DatabaseRequest::BlockHfInfoInRange(range) => get_blocks_hf_info_in_range(range, rpc)
|
||||
.instrument(span)
|
||||
.boxed(),
|
||||
DatabaseRequest::BlockWeightsInRange(range) => {
|
||||
get_blocks_weight_info_in_range(range, rpc).boxed()
|
||||
get_blocks_weight_info_in_range(range, rpc)
|
||||
.instrument(span)
|
||||
.boxed()
|
||||
}
|
||||
DatabaseRequest::BlockPOWInfoInRange(range) => {
|
||||
get_blocks_pow_info_in_range(range, rpc).boxed()
|
||||
DatabaseRequest::BlockPOWInfoInRange(range) => get_blocks_pow_info_in_range(range, rpc)
|
||||
.instrument(span)
|
||||
.boxed(),
|
||||
DatabaseRequest::BlockBatchInRange(range) => {
|
||||
get_blocks_in_range(range, rpc).instrument(span).boxed()
|
||||
}
|
||||
DatabaseRequest::BlockBatchInRange(range) => get_blocks_in_range(range, rpc).boxed(),
|
||||
}
|
||||
}
|
||||
}
|
||||
|
@ -334,20 +353,15 @@ async fn get_blocks_in_range<R: RpcConnection>(
|
|||
.map(|b| {
|
||||
Ok((
|
||||
monero_serai::block::Block::read(&mut b.block.as_slice())?,
|
||||
if let Some(txs) = b.txs {
|
||||
match txs {
|
||||
TransactionBlobs::Pruned(_) => {
|
||||
return Err("node sent pruned txs!".into())
|
||||
}
|
||||
TransactionBlobs::Normal(txs) => txs
|
||||
.into_iter()
|
||||
.map(|tx| {
|
||||
monero_serai::transaction::Transaction::read(&mut tx.as_slice())
|
||||
})
|
||||
.collect::<Result<_, _>>()?,
|
||||
}
|
||||
} else {
|
||||
vec![]
|
||||
match b.txs {
|
||||
TransactionBlobs::Pruned(_) => return Err("node sent pruned txs!".into()),
|
||||
TransactionBlobs::Normal(txs) => txs
|
||||
.into_iter()
|
||||
.map(|tx| {
|
||||
monero_serai::transaction::Transaction::read(&mut tx.as_slice())
|
||||
})
|
||||
.collect::<Result<_, _>>()?,
|
||||
TransactionBlobs::None => vec![],
|
||||
},
|
||||
))
|
||||
})
|
||||
|
@ -510,8 +524,3 @@ async fn get_blocks_hf_info_in_range<R: RpcConnection>(
|
|||
.collect(),
|
||||
))
|
||||
}
|
||||
|
||||
#[derive(Deserialize)]
|
||||
pub struct BResponse {
|
||||
pub blocks: Vec<BlockCompleteEntry>,
|
||||
}
|
||||
|
|
80
consensus/src/rpc/discover.rs
Normal file
80
consensus/src/rpc/discover.rs
Normal file
|
@ -0,0 +1,80 @@
|
|||
use std::collections::HashSet;
|
||||
use std::time::Duration;
|
||||
|
||||
use futures::channel::mpsc::SendError;
|
||||
use futures::stream::FuturesUnordered;
|
||||
use futures::{channel::mpsc, SinkExt, Stream, StreamExt, TryFutureExt, TryStream};
|
||||
use monero_serai::rpc::HttpRpc;
|
||||
use tokio::time::timeout;
|
||||
use tower::discover::Change;
|
||||
use tower::ServiceExt;
|
||||
use tracing::instrument;
|
||||
|
||||
use super::Rpc;
|
||||
use crate::Database;
|
||||
|
||||
#[instrument]
|
||||
async fn check_rpc(addr: String) -> Option<Rpc<HttpRpc>> {
|
||||
tracing::debug!("Sending request to node.");
|
||||
let rpc = HttpRpc::new(addr.clone()).ok()?;
|
||||
// make sure the RPC is actually reachable
|
||||
timeout(Duration::from_secs(2), rpc.get_height())
|
||||
.await
|
||||
.ok()?
|
||||
.ok()?;
|
||||
|
||||
tracing::debug!("Node sent ok response.");
|
||||
|
||||
Some(Rpc::new_http(addr))
|
||||
}
|
||||
|
||||
struct RPCDiscover<T> {
|
||||
rpc: T,
|
||||
initial_list: Vec<String>,
|
||||
ok_channel: mpsc::Sender<Change<usize, Rpc<HttpRpc>>>,
|
||||
already_connected: HashSet<String>,
|
||||
}
|
||||
|
||||
impl<T: Database> RPCDiscover<T> {
|
||||
async fn found_rpc(&mut self, rpc: Rpc<HttpRpc>) -> Result<(), SendError> {
|
||||
if self.already_connected.contains(&rpc.addr) {
|
||||
return Ok(());
|
||||
}
|
||||
|
||||
tracing::info!("Found node to connect to: {}", &rpc.addr);
|
||||
|
||||
let addr = rpc.addr.clone();
|
||||
self.ok_channel
|
||||
.send(Change::Insert(self.already_connected.len(), rpc))
|
||||
.await?;
|
||||
self.already_connected.insert(addr);
|
||||
|
||||
Ok(())
|
||||
}
|
||||
|
||||
pub async fn run(mut self) {
|
||||
loop {
|
||||
if !self.initial_list.is_empty() {
|
||||
let mut fut =
|
||||
FuturesUnordered::from_iter(self.initial_list.drain(..).map(check_rpc));
|
||||
|
||||
while let Some(res) = fut.next().await {
|
||||
if let Some(rpc) = res {
|
||||
if self.found_rpc(rpc).await.is_err() {
|
||||
tracing::info!("Stopping RPC discover channel closed!");
|
||||
return;
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
if self.already_connected.len() > 100 {
|
||||
tracing::info!("Stopping RPC discover, connected to 100 nodes!");
|
||||
}
|
||||
|
||||
tokio::time::sleep(Duration::from_secs(2)).await
|
||||
|
||||
// TODO: RPC request to get more peers
|
||||
}
|
||||
}
|
||||
}
|
|
@ -60,8 +60,11 @@ pub enum BucketError {
|
|||
#[error("Levin fragmented message was invalid: {0}")]
|
||||
InvalidFragmentedMessage(&'static str),
|
||||
/// Error decoding the body
|
||||
#[error("Error decoding bucket body")]
|
||||
BodyDecodingError(Box<dyn Debug>),
|
||||
#[error("Error decoding bucket body: {0}")]
|
||||
BodyDecodingError(Box<dyn std::error::Error>),
|
||||
/// The levin command is unknown
|
||||
#[error("The levin command is unknown")]
|
||||
UnknownCommand,
|
||||
/// I/O error
|
||||
#[error("I/O error: {0}")]
|
||||
IO(#[from] std::io::Error),
|
||||
|
|
|
@ -9,10 +9,10 @@ repository = "https://github.com/SyntheticBird45/cuprate/tree/main/net/monero-wi
|
|||
|
||||
[dependencies]
|
||||
levin-cuprate = {path="../levin"}
|
||||
epee-encoding = { git = "https://github.com/boog900/epee-encoding"}
|
||||
monero-epee-bin-serde = {git = "https://github.com/monero-rs/monero-epee-bin-serde.git", rev="e4a585a"}
|
||||
serde = {version = "1", features = ["derive"]}
|
||||
serde_with = "3"
|
||||
serde_bytes = "0.11"
|
||||
thiserror = "1"
|
||||
|
||||
[dev-dependencies]
|
||||
hex = "0.4.3"
|
||||
|
|
|
@ -29,12 +29,11 @@
|
|||
#![deny(unused_mut)]
|
||||
//#![deny(missing_docs)]
|
||||
|
||||
pub mod messages;
|
||||
pub mod network_address;
|
||||
pub mod p2p;
|
||||
mod serde_helpers;
|
||||
|
||||
pub use network_address::NetworkAddress;
|
||||
|
||||
pub use messages::*;
|
||||
pub use p2p::*;
|
||||
|
||||
pub type MoneroWireCodec = levin_cuprate::codec::LevinMessageCodec<Message>;
|
||||
|
|
|
@ -1,249 +0,0 @@
|
|||
// Rust Levin Library
|
||||
// Written in 2023 by
|
||||
// Cuprate Contributors
|
||||
//
|
||||
// Permission is hereby granted, free of charge, to any person obtaining a copy
|
||||
// of this software and associated documentation files (the "Software"), to deal
|
||||
// in the Software without restriction, including without limitation the rights
|
||||
// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell
|
||||
// copies of the Software, and to permit persons to whom the Software is
|
||||
// furnished to do so, subject to the following conditions:
|
||||
//
|
||||
// The above copyright notice and this permission notice shall be included in all
|
||||
// copies or substantial portions of the Software.
|
||||
//
|
||||
|
||||
//! This module defines a Monero `Message` enum which contains
|
||||
//! every possible Monero network message (levin body)
|
||||
|
||||
use levin_cuprate::{BucketBuilder, BucketError, LevinBody, MessageType};
|
||||
|
||||
pub mod admin;
|
||||
pub mod common;
|
||||
pub mod protocol;
|
||||
|
||||
pub use admin::{
|
||||
HandshakeRequest, HandshakeResponse, PingResponse, SupportFlagsResponse, TimedSyncRequest,
|
||||
TimedSyncResponse,
|
||||
};
|
||||
pub use common::{BasicNodeData, CoreSyncData, PeerListEntryBase};
|
||||
pub use protocol::{
|
||||
ChainRequest, ChainResponse, FluffyMissingTransactionsRequest, GetObjectsRequest,
|
||||
GetObjectsResponse, GetTxPoolCompliment, NewBlock, NewFluffyBlock, NewTransactions,
|
||||
};
|
||||
|
||||
pub enum ProtocolMessage {
|
||||
NewBlock(NewBlock),
|
||||
NewFluffyBlock(NewFluffyBlock),
|
||||
GetObjectsRequest(GetObjectsRequest),
|
||||
GetObjectsResponse(GetObjectsResponse),
|
||||
ChainRequest(ChainRequest),
|
||||
ChainEntryResponse(ChainResponse),
|
||||
NewTransactions(NewTransactions),
|
||||
FluffyMissingTransactionsRequest(FluffyMissingTransactionsRequest),
|
||||
GetTxPoolCompliment(GetTxPoolCompliment),
|
||||
}
|
||||
|
||||
impl ProtocolMessage {
|
||||
fn decode(buf: &[u8], command: u32) -> Result<Self, epee_encoding::Error> {
|
||||
Ok(match command {
|
||||
2001 => ProtocolMessage::NewBlock(epee_encoding::from_bytes(buf)?),
|
||||
2002 => ProtocolMessage::NewTransactions(epee_encoding::from_bytes(buf)?),
|
||||
2003 => ProtocolMessage::GetObjectsRequest(epee_encoding::from_bytes(buf)?),
|
||||
2004 => ProtocolMessage::GetObjectsResponse(epee_encoding::from_bytes(buf)?),
|
||||
2006 => ProtocolMessage::ChainRequest(epee_encoding::from_bytes(buf)?),
|
||||
2007 => ProtocolMessage::ChainEntryResponse(epee_encoding::from_bytes(buf)?),
|
||||
2008 => ProtocolMessage::NewFluffyBlock(epee_encoding::from_bytes(buf)?),
|
||||
2009 => {
|
||||
ProtocolMessage::FluffyMissingTransactionsRequest(epee_encoding::from_bytes(buf)?)
|
||||
}
|
||||
2010 => ProtocolMessage::GetTxPoolCompliment(epee_encoding::from_bytes(buf)?),
|
||||
_ => {
|
||||
return Err(epee_encoding::Error::Value(
|
||||
"Failed to decode message, unknown command",
|
||||
))
|
||||
}
|
||||
})
|
||||
}
|
||||
|
||||
fn build(&self, builder: &mut BucketBuilder) -> Result<(), epee_encoding::Error> {
|
||||
match self {
|
||||
ProtocolMessage::NewBlock(nb) => {
|
||||
builder.set_command(2001);
|
||||
builder.set_body(epee_encoding::to_bytes(nb)?);
|
||||
}
|
||||
ProtocolMessage::NewTransactions(nt) => {
|
||||
builder.set_command(2002);
|
||||
builder.set_body(epee_encoding::to_bytes(nt)?);
|
||||
}
|
||||
ProtocolMessage::GetObjectsRequest(gt) => {
|
||||
builder.set_command(2003);
|
||||
builder.set_body(epee_encoding::to_bytes(gt)?);
|
||||
}
|
||||
ProtocolMessage::GetObjectsResponse(ge) => {
|
||||
builder.set_command(2004);
|
||||
builder.set_body(epee_encoding::to_bytes(ge)?);
|
||||
}
|
||||
ProtocolMessage::ChainRequest(ct) => {
|
||||
builder.set_command(2006);
|
||||
builder.set_body(epee_encoding::to_bytes(ct)?);
|
||||
}
|
||||
ProtocolMessage::ChainEntryResponse(ce) => {
|
||||
builder.set_command(2007);
|
||||
builder.set_body(epee_encoding::to_bytes(ce)?);
|
||||
}
|
||||
ProtocolMessage::NewFluffyBlock(fb) => {
|
||||
builder.set_command(2008);
|
||||
builder.set_body(epee_encoding::to_bytes(fb)?);
|
||||
}
|
||||
ProtocolMessage::FluffyMissingTransactionsRequest(ft) => {
|
||||
builder.set_command(2009);
|
||||
builder.set_body(epee_encoding::to_bytes(ft)?);
|
||||
}
|
||||
ProtocolMessage::GetTxPoolCompliment(tp) => {
|
||||
builder.set_command(2010);
|
||||
builder.set_body(epee_encoding::to_bytes(tp)?);
|
||||
}
|
||||
}
|
||||
Ok(())
|
||||
}
|
||||
}
|
||||
|
||||
pub enum RequestMessage {
|
||||
Handshake(HandshakeRequest),
|
||||
Ping,
|
||||
SupportFlags,
|
||||
TimedSync(TimedSyncRequest),
|
||||
}
|
||||
|
||||
impl RequestMessage {
|
||||
fn decode(buf: &[u8], command: u32) -> Result<Self, epee_encoding::Error> {
|
||||
Ok(match command {
|
||||
1001 => RequestMessage::Handshake(epee_encoding::from_bytes(buf)?),
|
||||
1002 => RequestMessage::TimedSync(epee_encoding::from_bytes(buf)?),
|
||||
1003 => RequestMessage::Ping,
|
||||
1007 => RequestMessage::SupportFlags,
|
||||
_ => {
|
||||
return Err(epee_encoding::Error::Value(
|
||||
"Failed to decode message, unknown command",
|
||||
))
|
||||
}
|
||||
})
|
||||
}
|
||||
|
||||
fn build(&self, builder: &mut BucketBuilder) -> Result<(), epee_encoding::Error> {
|
||||
match self {
|
||||
RequestMessage::Handshake(hs) => {
|
||||
builder.set_command(1001);
|
||||
builder.set_body(epee_encoding::to_bytes(hs)?);
|
||||
}
|
||||
RequestMessage::TimedSync(ts) => {
|
||||
builder.set_command(1002);
|
||||
builder.set_body(epee_encoding::to_bytes(ts)?);
|
||||
}
|
||||
RequestMessage::Ping => {
|
||||
builder.set_command(1003);
|
||||
builder.set_body(Vec::new());
|
||||
}
|
||||
RequestMessage::SupportFlags => {
|
||||
builder.set_command(1007);
|
||||
builder.set_body(Vec::new());
|
||||
}
|
||||
}
|
||||
Ok(())
|
||||
}
|
||||
}
|
||||
|
||||
pub enum ResponseMessage {
|
||||
Handshake(HandshakeResponse),
|
||||
Ping(PingResponse),
|
||||
SupportFlags(SupportFlagsResponse),
|
||||
TimedSync(TimedSyncResponse),
|
||||
}
|
||||
|
||||
impl ResponseMessage {
|
||||
fn decode(buf: &[u8], command: u32) -> Result<Self, epee_encoding::Error> {
|
||||
Ok(match command {
|
||||
1001 => ResponseMessage::Handshake(epee_encoding::from_bytes(buf)?),
|
||||
1002 => ResponseMessage::TimedSync(epee_encoding::from_bytes(buf)?),
|
||||
1003 => ResponseMessage::Ping(epee_encoding::from_bytes(buf)?),
|
||||
1007 => ResponseMessage::SupportFlags(epee_encoding::from_bytes(buf)?),
|
||||
_ => {
|
||||
return Err(epee_encoding::Error::Value(
|
||||
"Failed to decode message, unknown command",
|
||||
))
|
||||
}
|
||||
})
|
||||
}
|
||||
|
||||
fn build(&self, builder: &mut BucketBuilder) -> Result<(), epee_encoding::Error> {
|
||||
match self {
|
||||
ResponseMessage::Handshake(hs) => {
|
||||
builder.set_command(1001);
|
||||
builder.set_body(epee_encoding::to_bytes(hs)?);
|
||||
}
|
||||
ResponseMessage::TimedSync(ts) => {
|
||||
builder.set_command(1002);
|
||||
builder.set_body(epee_encoding::to_bytes(ts)?);
|
||||
}
|
||||
ResponseMessage::Ping(pg) => {
|
||||
builder.set_command(1003);
|
||||
builder.set_body(epee_encoding::to_bytes(pg)?);
|
||||
}
|
||||
ResponseMessage::SupportFlags(sf) => {
|
||||
builder.set_command(1007);
|
||||
builder.set_body(epee_encoding::to_bytes(sf)?);
|
||||
}
|
||||
}
|
||||
Ok(())
|
||||
}
|
||||
}
|
||||
|
||||
pub enum Message {
|
||||
Request(RequestMessage),
|
||||
Response(ResponseMessage),
|
||||
Protocol(ProtocolMessage),
|
||||
}
|
||||
|
||||
impl LevinBody for Message {
|
||||
fn decode_message(body: &[u8], typ: MessageType, command: u32) -> Result<Self, BucketError> {
|
||||
Ok(match typ {
|
||||
MessageType::Request => Message::Request(
|
||||
RequestMessage::decode(body, command)
|
||||
.map_err(|e| BucketError::BodyDecodingError(Box::new(e)))?,
|
||||
),
|
||||
MessageType::Response => Message::Response(
|
||||
ResponseMessage::decode(body, command)
|
||||
.map_err(|e| BucketError::BodyDecodingError(Box::new(e)))?,
|
||||
),
|
||||
MessageType::Notification => Message::Protocol(
|
||||
ProtocolMessage::decode(body, command)
|
||||
.map_err(|e| BucketError::BodyDecodingError(Box::new(e)))?,
|
||||
),
|
||||
})
|
||||
}
|
||||
|
||||
fn encode(&self, builder: &mut BucketBuilder) -> Result<(), BucketError> {
|
||||
match self {
|
||||
Message::Protocol(pro) => {
|
||||
builder.set_message_type(MessageType::Notification);
|
||||
builder.set_return_code(0);
|
||||
pro.build(builder)
|
||||
.map_err(|e| BucketError::BodyDecodingError(Box::new(e)))?;
|
||||
}
|
||||
Message::Request(req) => {
|
||||
builder.set_message_type(MessageType::Request);
|
||||
builder.set_return_code(0);
|
||||
req.build(builder)
|
||||
.map_err(|e| BucketError::BodyDecodingError(Box::new(e)))?;
|
||||
}
|
||||
Message::Response(res) => {
|
||||
builder.set_message_type(MessageType::Response);
|
||||
builder.set_return_code(1);
|
||||
res.build(builder)
|
||||
.map_err(|e| BucketError::BodyDecodingError(Box::new(e)))?;
|
||||
}
|
||||
}
|
||||
Ok(())
|
||||
}
|
||||
}
|
|
@ -1,67 +0,0 @@
|
|||
use epee_encoding::{
|
||||
error::Error,
|
||||
io::{Read, Write},
|
||||
marker::InnerMarker,
|
||||
write_field, EpeeObject, EpeeObjectBuilder, EpeeValue,
|
||||
};
|
||||
|
||||
use super::{PrunedTxBlobEntry, TransactionBlobs};
|
||||
|
||||
impl EpeeObject for TransactionBlobs {
|
||||
type Builder = TransactionBlobsBuilder;
|
||||
fn number_of_fields(&self) -> u64 {
|
||||
1
|
||||
}
|
||||
fn write_fields<W: Write>(&self, w: &mut W) -> epee_encoding::error::Result<()> {
|
||||
match self {
|
||||
TransactionBlobs::Pruned(txs) => write_field(txs, "txs", w),
|
||||
TransactionBlobs::Normal(txs) => write_field(txs, "txs", w),
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
#[derive(Default)]
|
||||
pub enum TransactionBlobsBuilder {
|
||||
#[default]
|
||||
Init,
|
||||
Pruned(Vec<PrunedTxBlobEntry>),
|
||||
Normal(Vec<Vec<u8>>),
|
||||
}
|
||||
|
||||
impl EpeeObjectBuilder<TransactionBlobs> for TransactionBlobsBuilder {
|
||||
fn add_field<R: Read>(&mut self, name: &str, r: &mut R) -> epee_encoding::error::Result<bool> {
|
||||
if name != "txs" {
|
||||
return Ok(false);
|
||||
}
|
||||
|
||||
let marker = epee_encoding::read_marker(r)?;
|
||||
|
||||
if !marker.is_seq {
|
||||
return Err(Error::Format("Expected a sequence but got a single value"));
|
||||
}
|
||||
|
||||
match marker.inner_marker {
|
||||
InnerMarker::String => {
|
||||
let state = TransactionBlobsBuilder::Normal(Vec::<Vec<u8>>::read(r, &marker)?);
|
||||
let _ = std::mem::replace(self, state);
|
||||
}
|
||||
InnerMarker::Object => {
|
||||
let state =
|
||||
TransactionBlobsBuilder::Pruned(Vec::<PrunedTxBlobEntry>::read(r, &marker)?);
|
||||
let _ = std::mem::replace(self, state);
|
||||
}
|
||||
|
||||
_ => return Err(Error::Format("Unexpected marker")),
|
||||
}
|
||||
|
||||
Ok(true)
|
||||
}
|
||||
|
||||
fn finish(self) -> epee_encoding::error::Result<TransactionBlobs> {
|
||||
match self {
|
||||
TransactionBlobsBuilder::Init => Err(Error::Format("Required field was not in data")),
|
||||
TransactionBlobsBuilder::Normal(txs) => Ok(TransactionBlobs::Normal(txs)),
|
||||
TransactionBlobsBuilder::Pruned(txs) => Ok(TransactionBlobs::Pruned(txs)),
|
||||
}
|
||||
}
|
||||
}
|
|
@ -1,106 +0,0 @@
|
|||
use serde::de::{Error, SeqAccess};
|
||||
use serde::ser::SerializeSeq;
|
||||
use serde::{
|
||||
de::{Deserialize, Visitor},
|
||||
Deserializer, Serialize, Serializer,
|
||||
};
|
||||
use std::fmt::Formatter;
|
||||
|
||||
use super::TransactionBlobs;
|
||||
|
||||
impl<'de> Deserialize<'de> for TransactionBlobs {
|
||||
fn deserialize<D>(deserializer: D) -> Result<Self, D::Error>
|
||||
where
|
||||
D: Deserializer<'de>,
|
||||
{
|
||||
struct TBVisitor;
|
||||
|
||||
impl<'de> Visitor<'de> for TBVisitor {
|
||||
type Value = TransactionBlobs;
|
||||
|
||||
fn expecting(&self, formatter: &mut Formatter) -> std::fmt::Result {
|
||||
write!(formatter, "A sequence of transactions blob")
|
||||
}
|
||||
|
||||
fn visit_seq<A>(self, mut seq: A) -> Result<Self::Value, A::Error>
|
||||
where
|
||||
A: SeqAccess<'de>,
|
||||
{
|
||||
let mut normal = Vec::new();
|
||||
//let pruned = Vec::new();
|
||||
|
||||
while let Some(val) = seq.next_element::<SingleBlob>()? {
|
||||
match val {
|
||||
SingleBlob::Pruned(tx) => normal.push(tx),
|
||||
}
|
||||
}
|
||||
|
||||
Ok(TransactionBlobs::Normal(normal))
|
||||
}
|
||||
}
|
||||
|
||||
deserializer.deserialize_any(TBVisitor)
|
||||
}
|
||||
}
|
||||
|
||||
impl Serialize for TransactionBlobs {
|
||||
fn serialize<S>(&self, serializer: S) -> Result<S::Ok, S::Error>
|
||||
where
|
||||
S: Serializer,
|
||||
{
|
||||
match self {
|
||||
TransactionBlobs::Pruned(_) => todo!(),
|
||||
TransactionBlobs::Normal(txs) => {
|
||||
let mut seq_ser = serializer.serialize_seq(Some(txs.len()))?;
|
||||
for tx in txs {
|
||||
seq_ser.serialize_element(tx)?;
|
||||
}
|
||||
seq_ser.end()
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
enum SingleBlob {
|
||||
Pruned(Vec<u8>),
|
||||
}
|
||||
|
||||
impl<'de> Deserialize<'de> for SingleBlob {
|
||||
fn deserialize<D>(deserializer: D) -> Result<Self, D::Error>
|
||||
where
|
||||
D: Deserializer<'de>,
|
||||
{
|
||||
struct TBDSVisitor;
|
||||
|
||||
impl<'de> Visitor<'de> for TBDSVisitor {
|
||||
type Value = SingleBlob;
|
||||
|
||||
fn expecting(&self, formatter: &mut Formatter) -> std::fmt::Result {
|
||||
write!(formatter, "A single transaction blob")
|
||||
}
|
||||
|
||||
fn visit_bytes<E>(self, v: &[u8]) -> Result<Self::Value, E>
|
||||
where
|
||||
E: Error,
|
||||
{
|
||||
Ok(SingleBlob::Pruned(v.into()))
|
||||
}
|
||||
|
||||
fn visit_byte_buf<E>(self, v: Vec<u8>) -> Result<Self::Value, E>
|
||||
where
|
||||
E: Error,
|
||||
{
|
||||
Ok(SingleBlob::Pruned(v))
|
||||
}
|
||||
|
||||
fn visit_newtype_struct<D>(self, deserializer: D) -> Result<Self::Value, D::Error>
|
||||
where
|
||||
D: Deserializer<'de>,
|
||||
{
|
||||
todo!("Pruned blobs")
|
||||
}
|
||||
}
|
||||
|
||||
deserializer.deserialize_any(TBDSVisitor)
|
||||
}
|
||||
}
|
|
@ -20,7 +20,10 @@
|
|||
use std::net::{SocketAddrV4, SocketAddrV6};
|
||||
use std::{hash::Hash, net};
|
||||
|
||||
mod builder;
|
||||
use serde::{Deserialize, Serialize};
|
||||
|
||||
mod serde_helper;
|
||||
use serde_helper::*;
|
||||
|
||||
#[derive(Debug, PartialEq, Eq, Clone, Copy)]
|
||||
pub enum NetZone {
|
||||
|
@ -31,7 +34,9 @@ pub enum NetZone {
|
|||
|
||||
/// A network address which can be encoded into the format required
|
||||
/// to send to other Monero peers.
|
||||
#[derive(Clone, Copy, Debug, PartialEq, Eq, Hash)]
|
||||
#[derive(Clone, Copy, Debug, PartialEq, Eq, Hash, Serialize, Deserialize)]
|
||||
#[serde(try_from = "TaggedNetworkAddress")]
|
||||
#[serde(into = "TaggedNetworkAddress")]
|
||||
pub enum NetworkAddress {
|
||||
/// IPv4
|
||||
IPv4(SocketAddrV4),
|
||||
|
|
|
@ -1,167 +0,0 @@
|
|||
use std::net::{Ipv4Addr, Ipv6Addr, SocketAddrV4, SocketAddrV6};
|
||||
|
||||
use epee_encoding::{
|
||||
error::Error,
|
||||
io::{Read, Write},
|
||||
read_epee_value, write_field, EpeeObject, EpeeObjectBuilder, EpeeValue,
|
||||
};
|
||||
|
||||
use super::NetworkAddress;
|
||||
|
||||
impl EpeeObject for NetworkAddress {
|
||||
type Builder = NetworkAddressBuilder;
|
||||
|
||||
fn number_of_fields(&self) -> u64 {
|
||||
2
|
||||
}
|
||||
|
||||
fn write_fields<W: Write>(&self, w: &mut W) -> epee_encoding::error::Result<()> {
|
||||
match self {
|
||||
NetworkAddress::IPv4(ip) => {
|
||||
write_field(&1_u8, "type", w)?;
|
||||
let addr = NetworkAddressWriter {
|
||||
host: ("m_ip", &u32::from_be_bytes(ip.ip().octets())),
|
||||
port: ("m_port", ip.port()),
|
||||
};
|
||||
write_field(&addr, "addr", w)
|
||||
}
|
||||
NetworkAddress::IPv6(ip) => {
|
||||
write_field(&2_u8, "type", w)?;
|
||||
let addr = NetworkAddressWriter {
|
||||
host: ("addr", &ip.ip().octets()),
|
||||
port: ("m_port", ip.port()),
|
||||
};
|
||||
write_field(&addr, "addr", w)
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
struct NetworkAddressWriter<'a, T> {
|
||||
host: (&'static str, &'a T),
|
||||
port: (&'static str, u16),
|
||||
}
|
||||
|
||||
#[derive(Default)]
|
||||
struct NetworkAddressWBuilder;
|
||||
|
||||
impl<'a, T> EpeeObjectBuilder<NetworkAddressWriter<'a, T>> for NetworkAddressWBuilder {
|
||||
fn add_field<R: Read>(&mut self, _name: &str, _r: &mut R) -> epee_encoding::Result<bool> {
|
||||
panic!("Not used")
|
||||
}
|
||||
|
||||
fn finish(self) -> epee_encoding::Result<NetworkAddressWriter<'a, T>> {
|
||||
panic!("Not used")
|
||||
}
|
||||
}
|
||||
|
||||
impl<'a, T: EpeeValue> EpeeObject for NetworkAddressWriter<'a, T> {
|
||||
type Builder = NetworkAddressWBuilder;
|
||||
|
||||
fn number_of_fields(&self) -> u64 {
|
||||
2
|
||||
}
|
||||
|
||||
fn write_fields<W: Write>(&self, w: &mut W) -> epee_encoding::Result<()> {
|
||||
write_field(self.host.1, self.host.0, w)?;
|
||||
write_field(&self.port.1, self.port.0, w)
|
||||
}
|
||||
}
|
||||
#[derive(Default)]
|
||||
struct NetworkAddressBuilderIntermediate {
|
||||
m_ip: Option<u32>,
|
||||
addr: Option<[u8; 16]>,
|
||||
m_port: Option<u16>,
|
||||
port: Option<u16>,
|
||||
host_tor: Option<[u8; 63]>,
|
||||
host_i2p: Option<[u8; 61]>,
|
||||
}
|
||||
|
||||
impl EpeeObject for NetworkAddressBuilderIntermediate {
|
||||
type Builder = Self;
|
||||
|
||||
fn number_of_fields(&self) -> u64 {
|
||||
panic!("This is only used on deserialization")
|
||||
}
|
||||
|
||||
fn write_fields<W: Write>(&self, _w: &mut W) -> epee_encoding::error::Result<()> {
|
||||
panic!("This is only used on deserialization")
|
||||
}
|
||||
}
|
||||
|
||||
impl EpeeObjectBuilder<NetworkAddressBuilderIntermediate> for NetworkAddressBuilderIntermediate {
|
||||
fn add_field<R: Read>(&mut self, name: &str, r: &mut R) -> epee_encoding::error::Result<bool> {
|
||||
match name {
|
||||
"m_ip" => self.m_ip = Some(read_epee_value(r)?),
|
||||
"addr" => self.addr = Some(read_epee_value(r)?),
|
||||
"m_port" => self.m_port = Some(read_epee_value(r)?),
|
||||
"port" => self.port = Some(read_epee_value(r)?),
|
||||
"host" => {
|
||||
let host: Vec<u8> = read_epee_value(r)?;
|
||||
if host.len() == 63 {
|
||||
self.host_tor = Some(host.try_into().unwrap());
|
||||
} else if host.len() == 61 {
|
||||
self.host_i2p = Some(host.try_into().unwrap());
|
||||
}
|
||||
}
|
||||
_ => return Ok(false),
|
||||
}
|
||||
|
||||
Ok(true)
|
||||
}
|
||||
|
||||
fn finish(self) -> epee_encoding::error::Result<NetworkAddressBuilderIntermediate> {
|
||||
Ok(self)
|
||||
}
|
||||
}
|
||||
|
||||
#[derive(Default)]
|
||||
pub struct NetworkAddressBuilder {
|
||||
ty: Option<u8>,
|
||||
addr: Option<NetworkAddressBuilderIntermediate>,
|
||||
}
|
||||
|
||||
impl EpeeObjectBuilder<NetworkAddress> for NetworkAddressBuilder {
|
||||
fn add_field<R: Read>(&mut self, name: &str, r: &mut R) -> epee_encoding::error::Result<bool> {
|
||||
match name {
|
||||
"type" => self.ty = Some(read_epee_value(r)?),
|
||||
"addr" => self.addr = Some(read_epee_value(r)?),
|
||||
_ => return Ok(false),
|
||||
}
|
||||
|
||||
Ok(true)
|
||||
}
|
||||
fn finish(self) -> epee_encoding::error::Result<NetworkAddress> {
|
||||
let addr = self
|
||||
.addr
|
||||
.ok_or(Error::Format("Required field was not in data"))?;
|
||||
|
||||
Ok(
|
||||
match self
|
||||
.ty
|
||||
.ok_or(Error::Format("Required field was not in data"))?
|
||||
{
|
||||
1 => NetworkAddress::IPv4(SocketAddrV4::new(
|
||||
Ipv4Addr::from(
|
||||
addr.m_ip
|
||||
.ok_or(Error::Format("Required field was not in data"))?,
|
||||
),
|
||||
addr.m_port
|
||||
.ok_or(Error::Format("Required field was not in data"))?,
|
||||
)),
|
||||
2 => NetworkAddress::IPv6(SocketAddrV6::new(
|
||||
Ipv6Addr::from(
|
||||
addr.addr
|
||||
.ok_or(Error::Format("Required field was not in data"))?,
|
||||
),
|
||||
addr.m_port
|
||||
.ok_or(Error::Format("Required field was not in data"))?,
|
||||
0,
|
||||
0,
|
||||
)),
|
||||
// TODO: tor/ i2p addresses
|
||||
_ => return Err(Error::Value("Unsupported network address")),
|
||||
},
|
||||
)
|
||||
}
|
||||
}
|
195
net/monero-wire/src/p2p.rs
Normal file
195
net/monero-wire/src/p2p.rs
Normal file
|
@ -0,0 +1,195 @@
|
|||
// Rust Levin Library
|
||||
// Written in 2023 by
|
||||
// Cuprate Contributors
|
||||
//
|
||||
// Permission is hereby granted, free of charge, to any person obtaining a copy
|
||||
// of this software and associated documentation files (the "Software"), to deal
|
||||
// in the Software without restriction, including without limitation the rights
|
||||
// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell
|
||||
// copies of the Software, and to permit persons to whom the Software is
|
||||
// furnished to do so, subject to the following conditions:
|
||||
//
|
||||
// The above copyright notice and this permission notice shall be included in all
|
||||
// copies or substantial portions of the Software.
|
||||
//
|
||||
|
||||
//! This module defines a Monero `Message` enum which contains
|
||||
//! every possible Monero network message (levin body)
|
||||
|
||||
use levin_cuprate::{BucketBuilder, BucketError, LevinBody, MessageType};
|
||||
|
||||
pub mod admin;
|
||||
pub mod common;
|
||||
pub mod protocol;
|
||||
|
||||
use admin::*;
|
||||
pub use common::{BasicNodeData, CoreSyncData, PeerListEntryBase};
|
||||
use protocol::*;
|
||||
|
||||
fn decode_message<T: serde::de::DeserializeOwned, Ret>(
|
||||
ret: impl FnOnce(T) -> Ret,
|
||||
buf: &[u8],
|
||||
) -> Result<Ret, BucketError> {
|
||||
let t = monero_epee_bin_serde::from_bytes(buf)
|
||||
.map_err(|e| BucketError::BodyDecodingError(e.into()))?;
|
||||
Ok(ret(t))
|
||||
}
|
||||
|
||||
fn build_message<T: serde::Serialize>(
|
||||
id: u32,
|
||||
val: &T,
|
||||
builder: &mut BucketBuilder,
|
||||
) -> Result<(), BucketError> {
|
||||
builder.set_command(id);
|
||||
builder.set_body(
|
||||
monero_epee_bin_serde::to_bytes(val)
|
||||
.map_err(|e| BucketError::BodyDecodingError(e.into()))?,
|
||||
);
|
||||
Ok(())
|
||||
}
|
||||
|
||||
pub enum ProtocolMessage {
|
||||
NewBlock(NewBlock),
|
||||
NewFluffyBlock(NewFluffyBlock),
|
||||
GetObjectsRequest(GetObjectsRequest),
|
||||
GetObjectsResponse(GetObjectsResponse),
|
||||
ChainRequest(ChainRequest),
|
||||
ChainEntryResponse(ChainResponse),
|
||||
NewTransactions(NewTransactions),
|
||||
FluffyMissingTransactionsRequest(FluffyMissingTransactionsRequest),
|
||||
GetTxPoolCompliment(GetTxPoolCompliment),
|
||||
}
|
||||
|
||||
impl ProtocolMessage {
|
||||
fn decode(buf: &[u8], command: u32) -> Result<Self, BucketError> {
|
||||
Ok(match command {
|
||||
2001 => decode_message(ProtocolMessage::NewBlock, buf)?,
|
||||
2002 => decode_message(ProtocolMessage::NewTransactions, buf)?,
|
||||
2003 => decode_message(ProtocolMessage::GetObjectsRequest, buf)?,
|
||||
2004 => decode_message(ProtocolMessage::GetObjectsResponse, buf)?,
|
||||
2006 => decode_message(ProtocolMessage::ChainRequest, buf)?,
|
||||
2007 => decode_message(ProtocolMessage::ChainEntryResponse, buf)?,
|
||||
2008 => decode_message(ProtocolMessage::NewFluffyBlock, buf)?,
|
||||
2009 => decode_message(ProtocolMessage::FluffyMissingTransactionsRequest, buf)?,
|
||||
2010 => decode_message(ProtocolMessage::GetTxPoolCompliment, buf)?,
|
||||
_ => return Err(BucketError::UnknownCommand),
|
||||
})
|
||||
}
|
||||
|
||||
fn build(&self, builder: &mut BucketBuilder) -> Result<(), BucketError> {
|
||||
match self {
|
||||
ProtocolMessage::NewBlock(val) => build_message(2001, val, builder)?,
|
||||
ProtocolMessage::NewTransactions(val) => build_message(2002, val, builder)?,
|
||||
ProtocolMessage::GetObjectsRequest(val) => build_message(2003, val, builder)?,
|
||||
ProtocolMessage::GetObjectsResponse(val) => build_message(2004, val, builder)?,
|
||||
ProtocolMessage::ChainRequest(val) => build_message(2006, val, builder)?,
|
||||
ProtocolMessage::ChainEntryResponse(val) => build_message(2007, &val, builder)?,
|
||||
ProtocolMessage::NewFluffyBlock(val) => build_message(2008, val, builder)?,
|
||||
ProtocolMessage::FluffyMissingTransactionsRequest(val) => {
|
||||
build_message(2009, val, builder)?
|
||||
}
|
||||
ProtocolMessage::GetTxPoolCompliment(val) => build_message(2010, val, builder)?,
|
||||
}
|
||||
Ok(())
|
||||
}
|
||||
}
|
||||
|
||||
pub enum RequestMessage {
|
||||
Handshake(HandshakeRequest),
|
||||
Ping,
|
||||
SupportFlags,
|
||||
TimedSync(TimedSyncRequest),
|
||||
}
|
||||
|
||||
impl RequestMessage {
|
||||
fn decode(buf: &[u8], command: u32) -> Result<Self, BucketError> {
|
||||
Ok(match command {
|
||||
1001 => decode_message(RequestMessage::Handshake, buf)?,
|
||||
1002 => decode_message(RequestMessage::TimedSync, buf)?,
|
||||
1003 => RequestMessage::Ping,
|
||||
1007 => RequestMessage::SupportFlags,
|
||||
_ => return Err(BucketError::UnknownCommand),
|
||||
})
|
||||
}
|
||||
|
||||
fn build(&self, builder: &mut BucketBuilder) -> Result<(), BucketError> {
|
||||
match self {
|
||||
RequestMessage::Handshake(val) => build_message(1001, val, builder)?,
|
||||
RequestMessage::TimedSync(val) => build_message(1002, val, builder)?,
|
||||
RequestMessage::Ping => {
|
||||
builder.set_command(1003);
|
||||
builder.set_body(Vec::new());
|
||||
}
|
||||
RequestMessage::SupportFlags => {
|
||||
builder.set_command(1007);
|
||||
builder.set_body(Vec::new());
|
||||
}
|
||||
}
|
||||
Ok(())
|
||||
}
|
||||
}
|
||||
|
||||
pub enum ResponseMessage {
|
||||
Handshake(HandshakeResponse),
|
||||
Ping(PingResponse),
|
||||
SupportFlags(SupportFlagsResponse),
|
||||
TimedSync(TimedSyncResponse),
|
||||
}
|
||||
|
||||
impl ResponseMessage {
|
||||
fn decode(buf: &[u8], command: u32) -> Result<Self, BucketError> {
|
||||
Ok(match command {
|
||||
1001 => decode_message(ResponseMessage::Handshake, buf)?,
|
||||
1002 => decode_message(ResponseMessage::TimedSync, buf)?,
|
||||
1003 => decode_message(ResponseMessage::Ping, buf)?,
|
||||
1007 => decode_message(ResponseMessage::SupportFlags, buf)?,
|
||||
_ => return Err(BucketError::UnknownCommand),
|
||||
})
|
||||
}
|
||||
|
||||
fn build(&self, builder: &mut BucketBuilder) -> Result<(), BucketError> {
|
||||
match self {
|
||||
ResponseMessage::Handshake(val) => build_message(1001, val, builder)?,
|
||||
ResponseMessage::TimedSync(val) => build_message(1002, val, builder)?,
|
||||
ResponseMessage::Ping(val) => build_message(1003, val, builder)?,
|
||||
ResponseMessage::SupportFlags(val) => build_message(1007, val, builder)?,
|
||||
}
|
||||
Ok(())
|
||||
}
|
||||
}
|
||||
|
||||
pub enum Message {
|
||||
Request(RequestMessage),
|
||||
Response(ResponseMessage),
|
||||
Protocol(ProtocolMessage),
|
||||
}
|
||||
|
||||
impl LevinBody for Message {
|
||||
fn decode_message(body: &[u8], typ: MessageType, command: u32) -> Result<Self, BucketError> {
|
||||
Ok(match typ {
|
||||
MessageType::Request => Message::Request(RequestMessage::decode(body, command)?),
|
||||
MessageType::Response => Message::Response(ResponseMessage::decode(body, command)?),
|
||||
MessageType::Notification => Message::Protocol(ProtocolMessage::decode(body, command)?),
|
||||
})
|
||||
}
|
||||
|
||||
fn encode(&self, builder: &mut BucketBuilder) -> Result<(), BucketError> {
|
||||
match self {
|
||||
Message::Protocol(pro) => {
|
||||
builder.set_message_type(MessageType::Notification);
|
||||
builder.set_return_code(0);
|
||||
pro.build(builder)
|
||||
}
|
||||
Message::Request(req) => {
|
||||
builder.set_message_type(MessageType::Request);
|
||||
builder.set_return_code(0);
|
||||
req.build(builder)
|
||||
}
|
||||
Message::Response(res) => {
|
||||
builder.set_message_type(MessageType::Response);
|
||||
builder.set_return_code(1);
|
||||
res.build(builder)
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
|
@ -18,12 +18,12 @@
|
|||
//! Admin message requests must be responded to in order unlike
|
||||
//! protocol messages.
|
||||
|
||||
use epee_encoding::EpeeObject;
|
||||
use serde::{Deserialize, Serialize};
|
||||
|
||||
use super::common::{BasicNodeData, CoreSyncData, PeerListEntryBase, PeerSupportFlags};
|
||||
|
||||
/// A Handshake Request
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct HandshakeRequest {
|
||||
/// Basic Node Data
|
||||
pub node_data: BasicNodeData,
|
||||
|
@ -32,30 +32,31 @@ pub struct HandshakeRequest {
|
|||
}
|
||||
|
||||
/// A Handshake Response
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct HandshakeResponse {
|
||||
/// Basic Node Data
|
||||
pub node_data: BasicNodeData,
|
||||
/// Core Sync Data
|
||||
pub payload_data: CoreSyncData,
|
||||
/// PeerList
|
||||
#[epee_default(Vec::new())]
|
||||
#[serde(default = "Vec::new")]
|
||||
pub local_peerlist_new: Vec<PeerListEntryBase>,
|
||||
}
|
||||
|
||||
/// A TimedSync Request
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct TimedSyncRequest {
|
||||
/// Core Sync Data
|
||||
pub payload_data: CoreSyncData,
|
||||
}
|
||||
|
||||
/// A TimedSync Response
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct TimedSyncResponse {
|
||||
/// Core Sync Data
|
||||
pub payload_data: CoreSyncData,
|
||||
/// PeerList
|
||||
#[serde(default = "Vec::new")]
|
||||
pub local_peerlist_new: Vec<PeerListEntryBase>,
|
||||
}
|
||||
|
||||
|
@ -63,7 +64,7 @@ pub struct TimedSyncResponse {
|
|||
pub const PING_OK_RESPONSE_STATUS_TEXT: &str = "OK";
|
||||
|
||||
/// A Ping Response
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct PingResponse {
|
||||
/// Status: should be `PING_OK_RESPONSE_STATUS_TEXT`
|
||||
pub status: String,
|
||||
|
@ -72,10 +73,9 @@ pub struct PingResponse {
|
|||
}
|
||||
|
||||
/// A Support Flags Response
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct SupportFlagsResponse {
|
||||
/// Support Flags
|
||||
#[epee_try_from_into(u32)]
|
||||
pub support_flags: PeerSupportFlags,
|
||||
}
|
||||
|
||||
|
@ -83,7 +83,7 @@ pub struct SupportFlagsResponse {
|
|||
mod tests {
|
||||
|
||||
use super::{BasicNodeData, CoreSyncData, HandshakeRequest, HandshakeResponse};
|
||||
use crate::messages::common::PeerSupportFlags;
|
||||
use crate::p2p::common::PeerSupportFlags;
|
||||
|
||||
#[test]
|
||||
fn serde_handshake_req() {
|
||||
|
@ -101,7 +101,7 @@ mod tests {
|
|||
186, 15, 178, 70, 173, 170, 187, 31, 70, 50, 227, 11, 116, 111, 112, 95, 118, 101, 114,
|
||||
115, 105, 111, 110, 8, 1,
|
||||
];
|
||||
let handshake: HandshakeRequest = epee_encoding::from_bytes(&bytes).unwrap();
|
||||
let handshake: HandshakeRequest = monero_epee_bin_serde::from_bytes(&bytes).unwrap();
|
||||
let basic_node_data = BasicNodeData {
|
||||
my_port: 0,
|
||||
network_id: [
|
||||
|
@ -128,8 +128,9 @@ mod tests {
|
|||
assert_eq!(basic_node_data, handshake.node_data);
|
||||
assert_eq!(core_sync_data, handshake.payload_data);
|
||||
|
||||
let encoded_bytes = epee_encoding::to_bytes(&handshake).unwrap();
|
||||
let handshake_2: HandshakeRequest = epee_encoding::from_bytes(&encoded_bytes).unwrap();
|
||||
let encoded_bytes = monero_epee_bin_serde::to_bytes(&handshake).unwrap();
|
||||
let handshake_2: HandshakeRequest =
|
||||
monero_epee_bin_serde::from_bytes(&encoded_bytes).unwrap();
|
||||
|
||||
assert_eq!(handshake, handshake_2);
|
||||
}
|
||||
|
@ -905,7 +906,7 @@ mod tests {
|
|||
181, 216, 193, 135, 23, 186, 168, 207, 119, 86, 235, 11, 116, 111, 112, 95, 118, 101,
|
||||
114, 115, 105, 111, 110, 8, 16,
|
||||
];
|
||||
let handshake: HandshakeResponse = epee_encoding::from_bytes(&bytes).unwrap();
|
||||
let handshake: HandshakeResponse = monero_epee_bin_serde::from_bytes(&bytes).unwrap();
|
||||
|
||||
let basic_node_data = BasicNodeData {
|
||||
my_port: 18080,
|
||||
|
@ -934,8 +935,9 @@ mod tests {
|
|||
assert_eq!(core_sync_data, handshake.payload_data);
|
||||
assert_eq!(250, handshake.local_peerlist_new.len());
|
||||
|
||||
let encoded_bytes = epee_encoding::to_bytes(&handshake).unwrap();
|
||||
let handshake_2: HandshakeResponse = epee_encoding::from_bytes(&encoded_bytes).unwrap();
|
||||
let encoded_bytes = monero_epee_bin_serde::to_bytes(&handshake).unwrap();
|
||||
let handshake_2: HandshakeResponse =
|
||||
monero_epee_bin_serde::from_bytes(&encoded_bytes).unwrap();
|
||||
|
||||
assert_eq!(handshake, handshake_2);
|
||||
}
|
|
@ -14,20 +14,17 @@
|
|||
//
|
||||
|
||||
//! Common types that are used across multiple messages.
|
||||
//
|
||||
use epee_encoding::EpeeObject;
|
||||
|
||||
use serde::{Deserialize, Serialize};
|
||||
use serde_with::{serde_as, Bytes};
|
||||
use serde_bytes::ByteBuf;
|
||||
|
||||
use crate::{
|
||||
serde_helpers::{default_false, default_zero},
|
||||
NetworkAddress,
|
||||
};
|
||||
|
||||
mod builders;
|
||||
mod serde_impls;
|
||||
|
||||
#[derive(Debug, Clone, Copy, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, Copy, PartialEq, Eq, Serialize, Deserialize)]
|
||||
#[serde(transparent)]
|
||||
pub struct PeerSupportFlags(u32);
|
||||
|
||||
impl From<u32> for PeerSupportFlags {
|
||||
|
@ -70,7 +67,7 @@ impl From<u8> for PeerSupportFlags {
|
|||
}
|
||||
|
||||
/// Basic Node Data, information on the connected peer
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct BasicNodeData {
|
||||
/// Port
|
||||
pub my_port: u32,
|
||||
|
@ -80,39 +77,38 @@ pub struct BasicNodeData {
|
|||
pub peer_id: u64,
|
||||
/// The Peers Support Flags
|
||||
/// (If this is not in the message the default is 0)
|
||||
#[epee_try_from_into(u32)]
|
||||
#[epee_default(0_u32)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub support_flags: PeerSupportFlags,
|
||||
/// RPC Port
|
||||
/// (If this is not in the message the default is 0)
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub rpc_port: u16,
|
||||
/// RPC Credits Per Hash
|
||||
/// (If this is not in the message the default is 0)
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub rpc_credits_per_hash: u32,
|
||||
}
|
||||
|
||||
/// Core Sync Data, information on the sync state of a peer
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Deserialize, Serialize)]
|
||||
pub struct CoreSyncData {
|
||||
/// Cumulative Difficulty Low
|
||||
/// The lower 64 bits of the 128 bit cumulative difficulty
|
||||
pub cumulative_difficulty: u64,
|
||||
/// Cumulative Difficulty High
|
||||
/// The upper 64 bits of the 128 bit cumulative difficulty
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub cumulative_difficulty_top64: u64,
|
||||
/// Current Height of the peer
|
||||
pub current_height: u64,
|
||||
/// Pruning Seed of the peer
|
||||
/// (If this is not in the message the default is 0)
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub pruning_seed: u32,
|
||||
/// Hash of the top block
|
||||
pub top_id: [u8; 32],
|
||||
/// Version of the top block
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub top_version: u8,
|
||||
}
|
||||
|
||||
|
@ -145,23 +141,23 @@ impl CoreSyncData {
|
|||
|
||||
/// PeerListEntryBase, information kept on a peer which will be entered
|
||||
/// in a peer list/store.
|
||||
#[derive(Clone, Copy, EpeeObject, Debug, Eq, PartialEq)]
|
||||
#[derive(Clone, Copy, Debug, Eq, PartialEq, Serialize, Deserialize)]
|
||||
pub struct PeerListEntryBase {
|
||||
/// The Peer Address
|
||||
pub adr: NetworkAddress,
|
||||
/// The Peer ID
|
||||
pub id: u64,
|
||||
/// The last Time The Peer Was Seen
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub last_seen: i64,
|
||||
/// The Pruning Seed
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub pruning_seed: u32,
|
||||
/// The RPC port
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub rpc_port: u16,
|
||||
/// The RPC credits per hash
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub rpc_credits_per_hash: u32,
|
||||
}
|
||||
|
||||
|
@ -173,18 +169,21 @@ impl std::hash::Hash for PeerListEntryBase {
|
|||
}
|
||||
|
||||
/// A pruned tx with the hash of the missing prunable data
|
||||
#[derive(Clone, Debug, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Clone, Debug, PartialEq, Eq, Deserialize, Serialize)]
|
||||
pub struct PrunedTxBlobEntry {
|
||||
/// The Tx
|
||||
pub tx: Vec<u8>,
|
||||
pub tx: ByteBuf,
|
||||
/// The Prunable Tx Hash
|
||||
pub prunable_hash: [u8; 32],
|
||||
}
|
||||
|
||||
#[derive(Clone, Debug, PartialEq, Eq)]
|
||||
#[derive(Clone, Debug, PartialEq, Eq, Deserialize, Serialize)]
|
||||
#[serde(untagged)]
|
||||
pub enum TransactionBlobs {
|
||||
Pruned(Vec<PrunedTxBlobEntry>),
|
||||
Normal(Vec<Vec<u8>>),
|
||||
Normal(Vec<ByteBuf>),
|
||||
#[serde(skip_serializing)]
|
||||
None,
|
||||
}
|
||||
|
||||
impl TransactionBlobs {
|
||||
|
@ -192,33 +191,36 @@ impl TransactionBlobs {
|
|||
match self {
|
||||
TransactionBlobs::Normal(txs) => txs.len(),
|
||||
TransactionBlobs::Pruned(txs) => txs.len(),
|
||||
TransactionBlobs::None => 0,
|
||||
}
|
||||
}
|
||||
|
||||
pub fn is_empty(&self) -> bool {
|
||||
self.len() == 0
|
||||
}
|
||||
|
||||
fn none() -> TransactionBlobs {
|
||||
TransactionBlobs::None
|
||||
}
|
||||
}
|
||||
|
||||
/// A Block that can contain transactions
|
||||
#[serde_as]
|
||||
#[derive(Clone, Debug, EpeeObject, Serialize, Deserialize, PartialEq, Eq)]
|
||||
#[derive(Clone, Debug, Serialize, Deserialize, PartialEq, Eq)]
|
||||
pub struct BlockCompleteEntry {
|
||||
/// True if tx data is pruned
|
||||
#[epee_default(false)]
|
||||
#[serde(default = "default_false")]
|
||||
pub pruned: bool,
|
||||
/// The Block
|
||||
#[serde_as(as = "Bytes")]
|
||||
//#[serde_as(as = "Bytes")]
|
||||
#[serde(with = "serde_bytes")]
|
||||
pub block: Vec<u8>,
|
||||
/// The Block Weight/Size
|
||||
#[epee_default(0)]
|
||||
#[serde(default = "default_zero")]
|
||||
pub block_weight: u64,
|
||||
/// The blocks txs
|
||||
#[epee_default(None)]
|
||||
#[serde(skip_serializing_if = "Option::is_none")]
|
||||
pub txs: Option<TransactionBlobs>,
|
||||
#[serde(skip_serializing_if = "TransactionBlobs::is_empty")]
|
||||
#[serde(default = "TransactionBlobs::none")]
|
||||
pub txs: TransactionBlobs,
|
||||
}
|
||||
|
||||
#[cfg(test)]
|
|
@ -18,12 +18,13 @@
|
|||
//! Protocol message requests don't have to be responded to in order unlike
|
||||
//! admin messages.
|
||||
|
||||
use epee_encoding::EpeeObject;
|
||||
use serde::{Deserialize, Serialize};
|
||||
|
||||
use super::common::BlockCompleteEntry;
|
||||
use crate::serde_helpers::*;
|
||||
|
||||
/// A block that SHOULD have transactions
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct NewBlock {
|
||||
/// Block with txs
|
||||
pub b: BlockCompleteEntry,
|
||||
|
@ -32,51 +33,53 @@ pub struct NewBlock {
|
|||
}
|
||||
|
||||
/// New Tx Pool Transactions
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct NewTransactions {
|
||||
/// Tx Blobs
|
||||
pub txs: Vec<Vec<u8>>,
|
||||
/// Dandelionpp true if fluff - backwards compatible mode is fluff
|
||||
#[epee_default(true)]
|
||||
#[serde(default = "default_true")]
|
||||
pub dandelionpp_fluff: bool,
|
||||
/// Padding
|
||||
#[epee_alt_name("_")]
|
||||
#[serde(rename = "_")]
|
||||
pub padding: Vec<u8>,
|
||||
}
|
||||
|
||||
/// A Request For Blocks
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct GetObjectsRequest {
|
||||
/// Block hashes we want
|
||||
pub blocks: Vec<[u8; 32]>,
|
||||
/// Pruned
|
||||
#[epee_default(false)]
|
||||
#[serde(default = "default_false")]
|
||||
pub pruned: bool,
|
||||
}
|
||||
|
||||
/// A Blocks Response
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct GetObjectsResponse {
|
||||
/// Blocks
|
||||
// We dont need to give this a default value as there always is at least 1 block
|
||||
pub blocks: Vec<BlockCompleteEntry>,
|
||||
/// Missed IDs
|
||||
#[serde(default = "Vec::new")]
|
||||
pub missed_ids: Vec<[u8; 32]>,
|
||||
/// The height of the peers blockchain
|
||||
pub current_blockchain_height: u64,
|
||||
}
|
||||
|
||||
/// A Chain Request
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct ChainRequest {
|
||||
/// Block IDs
|
||||
pub block_ids: Vec<[u8; 32]>,
|
||||
/// Prune
|
||||
#[epee_default(false)]
|
||||
#[serde(default = "default_false")]
|
||||
pub prune: bool,
|
||||
}
|
||||
|
||||
/// A Chain Response
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct ChainResponse {
|
||||
/// Start Height
|
||||
pub start_height: u64,
|
||||
|
@ -123,7 +126,7 @@ impl ChainResponse {
|
|||
}
|
||||
|
||||
/// A Block that doesn't have transactions unless requested
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct NewFluffyBlock {
|
||||
/// Block which might have transactions
|
||||
pub b: BlockCompleteEntry,
|
||||
|
@ -132,7 +135,7 @@ pub struct NewFluffyBlock {
|
|||
}
|
||||
|
||||
/// A request for Txs we are missing from our TxPool
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct FluffyMissingTransactionsRequest {
|
||||
/// The Block we are missing the Txs in
|
||||
pub block_hash: [u8; 32],
|
||||
|
@ -143,9 +146,10 @@ pub struct FluffyMissingTransactionsRequest {
|
|||
}
|
||||
|
||||
/// TxPoolCompliment
|
||||
#[derive(Debug, Clone, EpeeObject, PartialEq, Eq)]
|
||||
#[derive(Debug, Clone, PartialEq, Eq, Serialize, Deserialize)]
|
||||
pub struct GetTxPoolCompliment {
|
||||
/// Tx Hashes
|
||||
#[serde(default = "Vec::new")]
|
||||
pub hashes: Vec<[u8; 32]>,
|
||||
}
|
||||
|
Loading…
Reference in a new issue