introduce index resolver

This commit is contained in:
mpostma 2021-09-24 11:53:11 +02:00
parent 5353be74c3
commit 42a6260b65
23 changed files with 833 additions and 193 deletions

View file

@ -3,19 +3,17 @@ use std::error::Error;
use meilisearch_error::{Code, ErrorCode};
use crate::index_controller::indexes::error::IndexActorError;
pub type Result<T> = std::result::Result<T, UpdateActorError>;
pub type Result<T> = std::result::Result<T, UpdateLoopError>;
#[derive(Debug, thiserror::Error)]
#[allow(clippy::large_enum_variant)]
pub enum UpdateActorError {
pub enum UpdateLoopError {
#[error("Update {0} not found.")]
UnexistingUpdate(u64),
#[error("Internal error: {0}")]
Internal(Box<dyn Error + Send + Sync + 'static>),
#[error("{0}")]
IndexActor(#[from] IndexActorError),
//#[error("{0}")]
//IndexActor(#[from] IndexActorError),
#[error(
"update store was shut down due to a fatal error, please check your logs for more info."
)]
@ -26,7 +24,7 @@ pub enum UpdateActorError {
PayloadError(#[from] actix_web::error::PayloadError),
}
impl<T> From<tokio::sync::mpsc::error::SendError<T>> for UpdateActorError
impl<T> From<tokio::sync::mpsc::error::SendError<T>> for UpdateLoopError
where T: Sync + Send + 'static + fmt::Debug
{
fn from(other: tokio::sync::mpsc::error::SendError<T>) -> Self {
@ -34,28 +32,28 @@ where T: Sync + Send + 'static + fmt::Debug
}
}
impl From<tokio::sync::oneshot::error::RecvError> for UpdateActorError {
impl From<tokio::sync::oneshot::error::RecvError> for UpdateLoopError {
fn from(other: tokio::sync::oneshot::error::RecvError) -> Self {
Self::Internal(Box::new(other))
}
}
internal_error!(
UpdateActorError: heed::Error,
UpdateLoopError: heed::Error,
std::io::Error,
serde_json::Error,
tokio::task::JoinError
);
impl ErrorCode for UpdateActorError {
impl ErrorCode for UpdateLoopError {
fn error_code(&self) -> Code {
match self {
UpdateActorError::UnexistingUpdate(_) => Code::NotFound,
UpdateActorError::Internal(_) => Code::Internal,
UpdateActorError::IndexActor(e) => e.error_code(),
UpdateActorError::FatalUpdateStoreError => Code::Internal,
UpdateActorError::InvalidPayload(_) => Code::BadRequest,
UpdateActorError::PayloadError(error) => match error {
UpdateLoopError::UnexistingUpdate(_) => Code::NotFound,
UpdateLoopError::Internal(_) => Code::Internal,
//UpdateLoopError::IndexActor(e) => e.error_code(),
UpdateLoopError::FatalUpdateStoreError => Code::Internal,
UpdateLoopError::InvalidPayload(_) => Code::BadRequest,
UpdateLoopError::PayloadError(error) => match error {
actix_web::error::PayloadError::Overflow => Code::PayloadTooLarge,
_ => Code::Internal,
},

View file

@ -21,25 +21,25 @@ use serde_json::{Map, Value};
use tokio::sync::mpsc;
use uuid::Uuid;
use self::error::{Result, UpdateActorError};
use self::error::{Result, UpdateLoopError};
pub use self::message::UpdateMsg;
use self::store::{UpdateStore, UpdateStoreInfo};
use crate::index_controller::update_file_store::UpdateFileStore;
use status::UpdateStatus;
use super::indexes::IndexHandlerSender;
use super::index_resolver::HardStateIndexResolver;
use super::{DocumentAdditionFormat, Payload, Update};
pub type UpdateSender = mpsc::Sender<UpdateMsg>;
pub fn create_update_handler(
index_sender: IndexHandlerSender,
index_resolver: Arc<HardStateIndexResolver>,
db_path: impl AsRef<Path>,
update_store_size: usize,
) -> anyhow::Result<UpdateSender> {
let path = db_path.as_ref().to_owned();
let (sender, receiver) = mpsc::channel(100);
let actor = UpdateLoop::new(update_store_size, receiver, path, index_sender)?;
let actor = UpdateLoop::new(update_store_size, receiver, path, index_resolver)?;
tokio::task::spawn_local(actor.run());
@ -100,7 +100,7 @@ pub struct UpdateLoop {
store: Arc<UpdateStore>,
inbox: Option<mpsc::Receiver<UpdateMsg>>,
update_file_store: UpdateFileStore,
index_handle: IndexHandlerSender,
index_resolver: Arc<HardStateIndexResolver>,
must_exit: Arc<AtomicBool>,
}
@ -109,7 +109,7 @@ impl UpdateLoop {
update_db_size: usize,
inbox: mpsc::Receiver<UpdateMsg>,
path: impl AsRef<Path>,
index_handle: IndexHandlerSender,
index_resolver: Arc<HardStateIndexResolver>,
) -> anyhow::Result<Self> {
let path = path.as_ref().to_owned();
std::fs::create_dir_all(&path)?;
@ -119,7 +119,7 @@ impl UpdateLoop {
let must_exit = Arc::new(AtomicBool::new(false));
let store = UpdateStore::open(options, &path, index_handle.clone(), must_exit.clone())?;
let store = UpdateStore::open(options, &path, index_resolver.clone(), must_exit.clone())?;
let inbox = Some(inbox);
@ -128,9 +128,9 @@ impl UpdateLoop {
Ok(Self {
store,
inbox,
index_handle,
must_exit,
update_file_store,
index_resolver,
})
}
@ -249,7 +249,7 @@ impl UpdateLoop {
tokio::task::spawn_blocking(move || {
let result = store
.meta(uuid, id)?
.ok_or(UpdateActorError::UnexistingUpdate(id))?;
.ok_or(UpdateLoopError::UnexistingUpdate(id))?;
Ok(result)
})
.await?
@ -263,18 +263,19 @@ impl UpdateLoop {
Ok(())
}
async fn handle_snapshot(&self, uuids: HashSet<Uuid>, path: PathBuf) -> Result<()> {
let index_handle = self.index_handle.clone();
let update_store = self.store.clone();
async fn handle_snapshot(&self, _uuids: HashSet<Uuid>,_pathh: PathBuf) -> Result<()> {
todo!()
//let index_handle = self.index_resolver.clone();
//let update_store = self.store.clone();
tokio::task::spawn_blocking(move || update_store.snapshot(&uuids, &path, index_handle))
.await??;
//tokio::task::spawn_blocking(move || update_store.snapshot(&uuids, &path, index_handle))
//.await??;
Ok(())
//Ok(())
}
async fn handle_dump(&self, uuids: HashSet<Uuid>, path: PathBuf) -> Result<()> {
let index_handle = self.index_handle.clone();
let index_handle = self.index_resolver.clone();
let update_store = self.store.clone();
tokio::task::spawn_blocking(move || -> Result<()> {

View file

@ -1,16 +1,11 @@
use std::{
collections::HashSet,
fs::{create_dir_all, File},
io::Write,
path::{Path, PathBuf},
};
use std::{collections::HashSet, fs::{create_dir_all, File}, io::Write, path::{Path, PathBuf}, sync::Arc};
use heed::RoTxn;
use serde::{Deserialize, Serialize};
use uuid::Uuid;
use super::{Result, State, UpdateStore};
use crate::index_controller::{indexes::{IndexHandlerSender, IndexMsg}, updates::{status::UpdateStatus}};
use crate::index_controller::{index_resolver::HardStateIndexResolver, updates::status::UpdateStatus};
#[derive(Serialize, Deserialize)]
struct UpdateEntry {
@ -23,7 +18,7 @@ impl UpdateStore {
&self,
uuids: &HashSet<Uuid>,
path: PathBuf,
handle: IndexHandlerSender,
handle: Arc<HardStateIndexResolver>,
) -> Result<()> {
let state_lock = self.state.write();
state_lock.swap(State::Dumping);
@ -171,13 +166,14 @@ impl UpdateStore {
}
async fn dump_indexes(
uuids: &HashSet<Uuid>,
handle: IndexHandlerSender,
path: impl AsRef<Path>,
_uuids: &HashSet<Uuid>,
_handle: Arc<HardStateIndexResolver>,
_path: impl AsRef<Path>,
) -> Result<()> {
for uuid in uuids {
IndexMsg::dump(&handle, *uuid, path.as_ref().to_owned()).await?;
}
todo!()
//for uuid in uuids {
//IndexMsg::dump(&handle, *uuid, path.as_ref().to_owned()).await?;
//}
Ok(())
//Ok(())
}

View file

@ -12,7 +12,6 @@ use std::{
};
use arc_swap::ArcSwap;
use futures::StreamExt;
use heed::types::{ByteSlice, OwnedType, SerdeJson};
use heed::zerocopy::U64;
use heed::{CompactionOption, Database, Env, EnvOpenOptions};
@ -30,7 +29,6 @@ use super::RegisterUpdate;
use super::error::Result;
use super::status::{Enqueued, Processing};
use crate::EnvSizer;
use crate::index_controller::indexes::{CONCURRENT_INDEX_MSG, IndexHandlerSender, IndexMsg};
use crate::index_controller::update_files_path;
use crate::index_controller::updates::*;
@ -148,7 +146,7 @@ impl UpdateStore {
pub fn open(
options: EnvOpenOptions,
path: impl AsRef<Path>,
index_handle: IndexHandlerSender,
index_resolver: Arc<HardStateIndexResolver>,
must_exit: Arc<AtomicBool>,
) -> anyhow::Result<Arc<Self>> {
let (update_store, mut notification_receiver) = Self::new(options, path)?;
@ -173,7 +171,7 @@ impl UpdateStore {
loop {
match update_store_weak.upgrade() {
Some(update_store) => {
let handler = index_handle.clone();
let handler = index_resolver.clone();
let res = tokio::task::spawn_blocking(move || {
update_store.process_pending_update(handler)
})
@ -286,7 +284,7 @@ impl UpdateStore {
/// Executes the user provided function on the next pending update (the one with the lowest id).
/// This is asynchronous as it let the user process the update with a read-only txn and
/// only writing the result meta to the processed-meta store *after* it has been processed.
fn process_pending_update(&self, index_handle: IndexHandlerSender) -> Result<Option<()>> {
fn process_pending_update(&self, index_resolver: Arc<HardStateIndexResolver>) -> Result<Option<()>> {
// Create a read transaction to be able to retrieve the pending update in order.
let rtxn = self.env.read_txn()?;
let first_meta = self.pending_queue.first(&rtxn)?;
@ -303,7 +301,7 @@ impl UpdateStore {
state.swap(State::Processing(index_uuid, processing.clone()));
let result =
self.perform_update(processing, index_handle, index_uuid, global_id);
self.perform_update(processing, index_resolver, index_uuid, global_id);
state.swap(State::Idle);
@ -316,18 +314,18 @@ impl UpdateStore {
fn perform_update(
&self,
processing: Processing,
index_handle: IndexHandlerSender,
index_resolver: Arc<HardStateIndexResolver>,
index_uuid: Uuid,
global_id: u64,
) -> Result<Option<()>> {
// Process the pending update using the provided user function.
let handle = Handle::current();
let update_id = processing.id();
let result =
match handle.block_on(IndexMsg::update(&index_handle, index_uuid, processing.clone())) {
Ok(result) => result,
Err(e) => Err(processing.fail(e)),
};
//IndexMsg::update(index_resolver, index_uuid, processing.clone()
let result = match handle.block_on(index_resolver.get_index_by_uuid(index_uuid)) {
Ok(index) => index.handle_update(processing),
Err(e) => Err(processing.fail(e)),
};
// Once the pending update have been successfully processed
// we must remove the content from the pending and processing stores and
@ -484,9 +482,9 @@ impl UpdateStore {
pub fn snapshot(
&self,
uuids: &HashSet<Uuid>,
_uuids: &HashSet<Uuid>,
path: impl AsRef<Path>,
handle: IndexHandlerSender,
handle: Arc<HardStateIndexResolver>,
) -> Result<()> {
let state_lock = self.state.write();
state_lock.swap(State::Snapshoting);
@ -522,22 +520,23 @@ impl UpdateStore {
//}
}
let path = &path.as_ref().to_path_buf();
let handle = &handle;
let _path = &path.as_ref().to_path_buf();
let _handle = &handle;
// Perform the snapshot of each index concurently. Only a third of the capabilities of
// the index actor at a time not to put too much pressure on the index actor
let mut stream = futures::stream::iter(uuids.iter())
.map(move |uuid| IndexMsg::snapshot(handle,*uuid, path.clone()))
.buffer_unordered(CONCURRENT_INDEX_MSG / 3);
todo!()
//let mut stream = futures::stream::iter(uuids.iter())
//.map(move |uuid| IndexMsg::snapshot(handle,*uuid, path.clone()))
//.buffer_unordered(CONCURRENT_INDEX_MSG / 3);
Handle::current().block_on(async {
while let Some(res) = stream.next().await {
res?;
}
Ok(()) as Result<()>
})?;
//Handle::current().block_on(async {
//while let Some(res) = stream.next().await {
//res?;
//}
//Ok(()) as Result<()>
//})?;
Ok(())
//Ok(())
}
pub fn get_info(&self) -> Result<UpdateStoreInfo> {