Skip to content

Commit

Permalink
Introduced chunk API (#1524)
Browse files Browse the repository at this point in the history
* Introduced `chunk` API

* Saving genesis chunks to make consistent output from chunks RPC
  • Loading branch information
frol committed Oct 23, 2019
1 parent 991a4a7 commit 28d4d1e
Show file tree
Hide file tree
Showing 12 changed files with 196 additions and 61 deletions.
39 changes: 29 additions & 10 deletions chain/chain/src/chain.rs
Original file line number Diff line number Diff line change
Expand Up @@ -7,6 +7,7 @@ use chrono::prelude::{DateTime, Utc};
use chrono::Duration;
use log::{debug, info};

use near_primitives::block::genesis_chunks;
use near_primitives::hash::{hash, CryptoHash};
use near_primitives::merkle::{merklize, verify_path};
use near_primitives::receipt::Receipt;
Expand Down Expand Up @@ -185,11 +186,15 @@ impl Chain {

// Get runtime initial state and create genesis block out of it.
let (state_store_update, state_roots) = runtime_adapter.genesis_state();
let genesis = Block::genesis(
let genesis_chunks = genesis_chunks(
state_roots.clone(),
chain_genesis.time,
runtime_adapter.num_shards(),
chain_genesis.gas_limit,
);
let genesis = Block::genesis(
genesis_chunks.iter().map(|chunk| chunk.header.clone()).collect(),
chain_genesis.time,
chain_genesis.gas_limit,
chain_genesis.gas_price,
chain_genesis.total_supply,
);
Expand Down Expand Up @@ -227,6 +232,9 @@ impl Chain {
}
Err(err) => match err.kind() {
ErrorKind::DBNotFoundErr(_) => {
for chunk in genesis_chunks {
store_update.save_chunk(&chunk.chunk_hash, chunk.clone());
}
runtime_adapter.add_validator_proposals(
CryptoHash::default(),
genesis.hash(),
Expand All @@ -247,7 +255,15 @@ impl Chain {
store_update.save_chunk_extra(
&genesis.hash(),
chunk_header.inner.shard_id,
ChunkExtra::new(state_root, vec![], 0, chain_genesis.gas_limit, 0, 0, 0),
ChunkExtra::new(
state_root,
vec![],
0,
chain_genesis.gas_limit,
0,
0,
0,
),
);
}

Expand Down Expand Up @@ -1652,11 +1668,14 @@ impl<'a> ChainUpdate<'a> {
self.chain_store_update.get_chunk_clone_from_header(&chunk_header)?;

let any_transaction_is_invalid = chunk.transactions.iter().any(|t| {
self.chain_store_update.get_chain_store().check_blocks_on_same_chain(
&block.header,
&t.transaction.block_hash,
self.transaction_validity_period,
).is_err()
self.chain_store_update
.get_chain_store()
.check_blocks_on_same_chain(
&block.header,
&t.transaction.block_hash,
self.transaction_validity_period,
)
.is_err()
});
if any_transaction_is_invalid {
debug!(target: "chain", "Invalid transactions in the chunk: {:?}", chunk.transactions);
Expand Down Expand Up @@ -1693,7 +1712,7 @@ impl<'a> ChainUpdate<'a> {
gas_limit,
apply_result.total_rent_paid,
apply_result.total_validator_reward,
apply_result.total_balance_burnt
apply_result.total_balance_burnt,
),
);
// Save resulting receipts.
Expand Down Expand Up @@ -2200,7 +2219,7 @@ impl<'a> ChainUpdate<'a> {
gas_limit,
apply_result.total_rent_paid,
apply_result.total_validator_reward,
apply_result.total_balance_burnt
apply_result.total_balance_burnt,
);
self.chain_store_update.save_chunk_extra(&block_header.hash, shard_id, chunk_extra);

Expand Down
9 changes: 7 additions & 2 deletions chain/chain/src/types.rs
Original file line number Diff line number Diff line change
Expand Up @@ -483,17 +483,22 @@ mod tests {
use chrono::Utc;

use near_crypto::{InMemorySigner, KeyType, Signature};
use near_primitives::block::genesis_chunks;

use super::*;

#[test]
fn test_block_produce() {
let num_shards = 32;
let genesis = Block::genesis(
let genesis_chunks = genesis_chunks(
vec![StateRoot { hash: CryptoHash::default(), num_parts: 9 /* TODO MOO */ }],
Utc::now(),
num_shards,
1_000_000,
);
let genesis = Block::genesis(
genesis_chunks.into_iter().map(|chunk| chunk.header).collect(),
Utc::now(),
1_000_000,
100,
1_000_000_000,
);
Expand Down
1 change: 1 addition & 0 deletions chain/client/src/types.rs
Original file line number Diff line number Diff line change
Expand Up @@ -255,6 +255,7 @@ impl Message for GetBlock {

/// Actor message requesting a chunk by chunk hash and block hash + shard id.
pub enum GetChunk {
BlockHeight(BlockIndex, ShardId),
BlockHash(CryptoHash, ShardId),
ChunkHash(ChunkHash),
}
Expand Down
7 changes: 7 additions & 0 deletions chain/client/src/view_client.rs
Original file line number Diff line number Diff line change
Expand Up @@ -174,6 +174,13 @@ impl Handler<GetChunk> for ViewClientActor {
.get_chunk(&block.chunks[shard_id as usize].chunk_hash())
.map(Clone::clone)
})
},
GetChunk::BlockHeight(block_height, shard_id) => {
self.chain.get_block_by_height(block_height).map(Clone::clone).and_then(|block| {
self.chain
.get_chunk(&block.chunks[shard_id as usize].chunk_hash())
.map(Clone::clone)
})
}
}
.map(|chunk| chunk.into())
Expand Down
12 changes: 10 additions & 2 deletions chain/jsonrpc/client/src/lib.rs
Original file line number Diff line number Diff line change
Expand Up @@ -6,9 +6,9 @@ use serde::Deserialize;
use serde::Serialize;

use near_primitives::hash::CryptoHash;
use near_primitives::types::BlockIndex;
use near_primitives::types::{BlockIndex, ShardId};
use near_primitives::views::{
BlockView, ExecutionOutcomeView, FinalExecutionOutcomeView, QueryResponse, StatusResponse,
BlockView, ChunkView, ExecutionOutcomeView, FinalExecutionOutcomeView, QueryResponse, StatusResponse,
};

use crate::message::{from_slice, Message};
Expand All @@ -22,6 +22,13 @@ pub enum BlockId {
Hash(CryptoHash),
}

#[derive(Debug, Clone, PartialEq, Serialize, Deserialize)]
#[serde(untagged)]
pub enum ChunkId {
BlockShardId(BlockId, ShardId),
Hash(CryptoHash),
}

/// Timeout for establishing connection.
const CONNECT_TIMEOUT: Duration = Duration::from_secs(10);

Expand Down Expand Up @@ -180,6 +187,7 @@ jsonrpc_client!(pub struct JsonRpcClient {
pub fn tx(&mut self, hash: String) -> RpcRequest<FinalExecutionOutcomeView>;
pub fn tx_details(&mut self, hash: String) -> RpcRequest<ExecutionOutcomeView>;
pub fn block(&mut self, id: BlockId) -> RpcRequest<BlockView>;
pub fn chunk(&mut self, id: ChunkId) -> RpcRequest<ChunkView>;
});

/// Create new JSON RPC client that connects to the given address.
Expand Down
21 changes: 19 additions & 2 deletions chain/jsonrpc/src/lib.rs
Original file line number Diff line number Diff line change
Expand Up @@ -19,10 +19,10 @@ use async_utils::{delay, timeout};
use message::Message;
use message::{Request, RpcError};
use near_client::{
ClientActor, GetBlock, GetNetworkInfo, Query, Status, TxDetails, TxStatus, ViewClientActor,
ClientActor, GetBlock, GetChunk, GetNetworkInfo, Query, Status, TxDetails, TxStatus, ViewClientActor,
};
pub use near_jsonrpc_client as client;
use near_jsonrpc_client::{message, BlockId};
use near_jsonrpc_client::{message, BlockId, ChunkId};
use near_metrics::{Encoder, TextEncoder};
use near_network::{NetworkClientMessages, NetworkClientResponses};
use near_primitives::hash::CryptoHash;
Expand Down Expand Up @@ -143,6 +143,7 @@ impl JsonRpcHandler {
"tx" => self.tx_status(request.params).await,
"tx_details" => self.tx_details(request.params).await,
"block" => self.block(request.params).await,
"chunk" => self.chunk(request.params).await,
"network_info" => self.network_info().await,
_ => Err(RpcError::method_not_found(request.method)),
}
Expand Down Expand Up @@ -239,6 +240,22 @@ impl JsonRpcHandler {
)
}

async fn chunk(&self, params: Option<Value>) -> Result<Value, RpcError> {
let (chunk_id,) = parse_params::<(ChunkId,)>(params)?;
jsonify(
self.view_client_addr
.send(match chunk_id {
ChunkId::BlockShardId(block_id, shard_id) => match block_id {
BlockId::Height(block_height) => GetChunk::BlockHeight(block_height, shard_id),
BlockId::Hash(block_hash) => GetChunk::BlockHash(block_hash.into(), shard_id),
},
ChunkId::Hash(chunk_hash) => GetChunk::ChunkHash(chunk_hash.into()),
})
.compat()
.await,
)
}

async fn network_info(&self) -> Result<Value, RpcError> {
jsonify(self.client_addr.send(GetNetworkInfo {}).compat().await)
}
Expand Down
53 changes: 52 additions & 1 deletion chain/jsonrpc/tests/rpc_query.rs
Original file line number Diff line number Diff line change
Expand Up @@ -4,11 +4,13 @@ use actix::System;
use futures::future;
use futures::future::Future;

use near_crypto::Signature;
use near_jsonrpc::client::new_client;
use near_jsonrpc::test_utils::start_all;
use near_jsonrpc_client::BlockId;
use near_jsonrpc_client::{BlockId, ChunkId};
use near_primitives::hash::CryptoHash;
use near_primitives::test_utils::init_test_logger;
use near_primitives::types::ShardId;

/// Retrieve blocks via json rpc
#[test]
Expand Down Expand Up @@ -65,6 +67,55 @@ fn test_block_by_hash() {
.unwrap();
}

/// Retrieve blocks via json rpc
#[test]
fn test_chunk_by_hash() {
init_test_logger();

System::run(|| {
let (_view_client_addr, addr) = start_all(true);

let mut client = new_client(&format!("http://{}", addr.clone()));
actix::spawn(
client.chunk(ChunkId::BlockShardId(BlockId::Height(0), ShardId::from(0u64))).then(
move |chunk| {
let chunk = chunk.unwrap();
assert_eq!(chunk.header.balance_burnt, 0);
assert_eq!(chunk.header.chunk_hash.as_ref().len(), 32);
assert_eq!(chunk.header.encoded_length, 8);
assert_eq!(chunk.header.encoded_merkle_root.as_ref().len(), 32);
assert_eq!(chunk.header.gas_limit, 1000000);
assert_eq!(chunk.header.gas_used, 0);
assert_eq!(chunk.header.height_created, 0);
assert_eq!(chunk.header.height_included, 0);
assert_eq!(chunk.header.outgoing_receipts_root.as_ref().len(), 32);
assert_eq!(chunk.header.prev_block_hash.as_ref().len(), 32);
assert_eq!(chunk.header.prev_state_num_parts, 17);
assert_eq!(chunk.header.prev_state_root_hash.as_ref().len(), 32);
assert_eq!(chunk.header.rent_paid, 0);
assert_eq!(chunk.header.shard_id, 0);
assert!(if let Signature::ED25519(_) = chunk.header.signature {
true
} else {
false
});
assert_eq!(chunk.header.tx_root.as_ref(), &[0; 32]);
assert_eq!(chunk.header.validator_proposals, vec![]);
assert_eq!(chunk.header.validator_reward, 0);
let mut client = new_client(&format!("http://{}", addr));
client.chunk(ChunkId::Hash(chunk.header.chunk_hash)).then(move |same_chunk| {
let same_chunk = same_chunk.unwrap();
assert_eq!(chunk.header.chunk_hash, same_chunk.header.chunk_hash);
System::current().stop();
future::ok(())
})
},
),
);
})
.unwrap();
}

/// Connect to json rpc and query the client.
#[test]
fn test_query() {
Expand Down
10 changes: 7 additions & 3 deletions core/primitives/benches/serialization.rs
Original file line number Diff line number Diff line change
Expand Up @@ -9,7 +9,7 @@ use chrono::Utc;

use near_crypto::{InMemorySigner, KeyType, PublicKey, Signature};
use near_primitives::account::Account;
use near_primitives::block::Block;
use near_primitives::block::{genesis_chunks, Block};
use near_primitives::hash::CryptoHash;
use near_primitives::transaction::{Action, SignedTransaction, Transaction, TransferAction};
use near_primitives::types::{EpochId, StateRoot};
Expand All @@ -33,11 +33,15 @@ fn create_transaction() -> SignedTransaction {
}

fn create_block() -> Block {
let genesis = Block::genesis(
let genesis_chunks = genesis_chunks(
vec![StateRoot { hash: CryptoHash::default(), num_parts: 1 /* TODO MOO */ }],
Utc::now(),
1,
1_000,
);
let genesis = Block::genesis(
genesis_chunks.into_iter().map(|chunk| chunk.header).collect(),
Utc::now(),
1_000,
1_000,
1_000,
);
Expand Down
60 changes: 35 additions & 25 deletions core/primitives/src/block.rs
Original file line number Diff line number Diff line change
Expand Up @@ -7,7 +7,7 @@ use near_crypto::{EmptySigner, KeyType, PublicKey, Signature, Signer};

use crate::hash::{hash, CryptoHash};
use crate::merkle::merklize;
use crate::sharding::{ChunkHashHeight, ShardChunkHeader};
use crate::sharding::{ChunkHashHeight, EncodedShardChunk, ShardChunk, ShardChunkHeader};
use crate::types::{
Balance, BlockIndex, EpochId, Gas, MerkleHash, ShardId, StateRoot, ValidatorStake,
};
Expand Down Expand Up @@ -242,38 +242,48 @@ pub struct Block {
pub chunks: Vec<ShardChunkHeader>,
}

pub fn genesis_chunks(
state_roots: Vec<StateRoot>,
num_shards: ShardId,
initial_gas_limit: Gas,
) -> Vec<ShardChunk> {
assert!(state_roots.len() == 1 || state_roots.len() == (num_shards as usize));
(0..num_shards)
.map(|i| {
let (encoded_chunk, _) = EncodedShardChunk::new(
CryptoHash::default(),
state_roots[i as usize % state_roots.len()].clone(),
0,
i,
3,
1,
0,
initial_gas_limit,
0,
0,
0,
CryptoHash::default(),
vec![],
&vec![],
&vec![],
CryptoHash::default(),
&EmptySigner {},
)
.expect("Failed to decode genesis chunk");
encoded_chunk.decode_chunk(1).expect("Failed to decode genesis chunk")
})
.collect()
}

impl Block {
/// Returns genesis block for given genesis date and state root.
pub fn genesis(
state_roots: Vec<StateRoot>,
chunks: Vec<ShardChunkHeader>,
timestamp: DateTime<Utc>,
num_shards: ShardId,
initial_gas_limit: Gas,
initial_gas_price: Balance,
initial_total_supply: Balance,
) -> Self {
assert!(state_roots.len() == 1 || state_roots.len() == (num_shards as usize));
let chunks = (0..num_shards)
.map(|i| {
ShardChunkHeader::new(
CryptoHash::default(),
state_roots[i as usize % state_roots.len()].clone(),
CryptoHash::default(),
0,
0,
i,
0,
initial_gas_limit,
0,
0,
0,
CryptoHash::default(),
CryptoHash::default(),
vec![],
&EmptySigner {},
)
})
.collect();
Block {
header: BlockHeader::genesis(
Block::compute_state_root(&chunks),
Expand Down
Loading

0 comments on commit 28d4d1e

Please sign in to comment.